001    /**
002     * Licensed to the Apache Software Foundation (ASF) under one
003     * or more contributor license agreements.  See the NOTICE file
004     * distributed with this work for additional information
005     * regarding copyright ownership.  The ASF licenses this file
006     * to you under the Apache License, Version 2.0 (the
007     * "License"); you may not use this file except in compliance
008     * with the License.  You may obtain a copy of the License at
009     *
010     *     http://www.apache.org/licenses/LICENSE-2.0
011     *
012     * Unless required by applicable law or agreed to in writing, software
013     * distributed under the License is distributed on an "AS IS" BASIS,
014     * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
015     * See the License for the specific language governing permissions and
016     * limitations under the License.
017     */
018    
019    package org.apache.hadoop.hdfs;
020    
021    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_HA_NAMENODES_KEY_PREFIX;
022    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_HA_NAMENODE_ID_KEY;
023    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_BACKUP_ADDRESS_KEY;
024    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_HTTPS_ADDRESS_DEFAULT;
025    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_HTTPS_ADDRESS_KEY;
026    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_HTTP_ADDRESS_DEFAULT;
027    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_HTTP_ADDRESS_KEY;
028    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_RPC_ADDRESS_KEY;
029    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_SECONDARY_HTTP_ADDRESS_KEY;
030    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMENODE_SERVICE_RPC_ADDRESS_KEY;
031    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMESERVICES;
032    import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_NAMESERVICE_ID;
033    
034    import java.io.IOException;
035    import java.io.PrintStream;
036    import java.io.UnsupportedEncodingException;
037    import java.net.InetSocketAddress;
038    import java.net.URI;
039    import java.net.URISyntaxException;
040    import java.security.SecureRandom;
041    import java.util.Collection;
042    import java.util.Collections;
043    import java.util.Comparator;
044    import java.util.HashMap;
045    import java.util.HashSet;
046    import java.util.List;
047    import java.util.Map;
048    import java.util.Random;
049    import java.util.Set;
050    import java.util.concurrent.TimeUnit;
051    
052    import javax.net.SocketFactory;
053    
054    import org.apache.commons.cli.CommandLine;
055    import org.apache.commons.cli.CommandLineParser;
056    import org.apache.commons.cli.Option;
057    import org.apache.commons.cli.Options;
058    import org.apache.commons.cli.ParseException;
059    import org.apache.commons.cli.PosixParser;
060    import org.apache.commons.logging.Log;
061    import org.apache.commons.logging.LogFactory;
062    import org.apache.hadoop.HadoopIllegalArgumentException;
063    import org.apache.hadoop.classification.InterfaceAudience;
064    import org.apache.hadoop.conf.Configuration;
065    import org.apache.hadoop.fs.BlockLocation;
066    import org.apache.hadoop.fs.FileSystem;
067    import org.apache.hadoop.fs.Path;
068    import org.apache.hadoop.hdfs.protocol.AlreadyBeingCreatedException;
069    import org.apache.hadoop.hdfs.protocol.ClientDatanodeProtocol;
070    import org.apache.hadoop.hdfs.protocol.DatanodeID;
071    import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
072    import org.apache.hadoop.hdfs.protocol.HdfsConstants;
073    import org.apache.hadoop.hdfs.protocol.LocatedBlock;
074    import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
075    import org.apache.hadoop.hdfs.protocolPB.ClientDatanodeProtocolTranslatorPB;
076    import org.apache.hadoop.hdfs.server.namenode.FSDirectory;
077    import org.apache.hadoop.hdfs.server.namenode.NameNode;
078    import org.apache.hadoop.io.retry.RetryPolicies;
079    import org.apache.hadoop.io.retry.RetryPolicy;
080    import org.apache.hadoop.io.retry.RetryProxy;
081    import org.apache.hadoop.ipc.ProtobufRpcEngine;
082    import org.apache.hadoop.ipc.RPC;
083    import org.apache.hadoop.net.NetUtils;
084    import org.apache.hadoop.net.NodeBase;
085    import org.apache.hadoop.security.SecurityUtil;
086    import org.apache.hadoop.security.UserGroupInformation;
087    import org.apache.hadoop.util.StringUtils;
088    import org.apache.hadoop.util.ToolRunner;
089    
090    import com.google.common.base.Charsets;
091    import com.google.common.base.Joiner;
092    import com.google.common.base.Preconditions;
093    import com.google.common.collect.Lists;
094    import com.google.common.collect.Maps;
095    import com.google.common.primitives.SignedBytes;
096    import com.google.protobuf.BlockingService;
097    
098    @InterfaceAudience.Private
099    public class DFSUtil {
100      public static final Log LOG = LogFactory.getLog(DFSUtil.class.getName());
101      
102      public static final byte[] EMPTY_BYTES = {};
103    
104      /** Compare two byte arrays by lexicographical order. */
105      public static int compareBytes(byte[] left, byte[] right) {
106        if (left == null) {
107          left = EMPTY_BYTES;
108        }
109        if (right == null) {
110          right = EMPTY_BYTES;
111        }
112        return SignedBytes.lexicographicalComparator().compare(left, right);
113      }
114    
115      private DFSUtil() { /* Hidden constructor */ }
116      private static final ThreadLocal<Random> RANDOM = new ThreadLocal<Random>() {
117        @Override
118        protected Random initialValue() {
119          return new Random();
120        }
121      };
122      
123      private static final ThreadLocal<SecureRandom> SECURE_RANDOM = new ThreadLocal<SecureRandom>() {
124        @Override
125        protected SecureRandom initialValue() {
126          return new SecureRandom();
127        }
128      };
129    
130      /** @return a pseudo random number generator. */
131      public static Random getRandom() {
132        return RANDOM.get();
133      }
134      
135      /** @return a pseudo secure random number generator. */
136      public static SecureRandom getSecureRandom() {
137        return SECURE_RANDOM.get();
138      }
139    
140      /**
141       * Compartor for sorting DataNodeInfo[] based on decommissioned states.
142       * Decommissioned nodes are moved to the end of the array on sorting with
143       * this compartor.
144       */
145      public static final Comparator<DatanodeInfo> DECOM_COMPARATOR = 
146        new Comparator<DatanodeInfo>() {
147          @Override
148          public int compare(DatanodeInfo a, DatanodeInfo b) {
149            return a.isDecommissioned() == b.isDecommissioned() ? 0 : 
150              a.isDecommissioned() ? 1 : -1;
151          }
152        };
153        
154          
155      /**
156       * Comparator for sorting DataNodeInfo[] based on decommissioned/stale states.
157       * Decommissioned/stale nodes are moved to the end of the array on sorting
158       * with this comparator.
159       */ 
160      @InterfaceAudience.Private 
161      public static class DecomStaleComparator implements Comparator<DatanodeInfo> {
162        private long staleInterval;
163    
164        /**
165         * Constructor of DecomStaleComparator
166         * 
167         * @param interval
168         *          The time interval for marking datanodes as stale is passed from
169         *          outside, since the interval may be changed dynamically
170         */
171        public DecomStaleComparator(long interval) {
172          this.staleInterval = interval;
173        }
174    
175        @Override
176        public int compare(DatanodeInfo a, DatanodeInfo b) {
177          // Decommissioned nodes will still be moved to the end of the list
178          if (a.isDecommissioned()) {
179            return b.isDecommissioned() ? 0 : 1;
180          } else if (b.isDecommissioned()) {
181            return -1;
182          }
183          // Stale nodes will be moved behind the normal nodes
184          boolean aStale = a.isStale(staleInterval);
185          boolean bStale = b.isStale(staleInterval);
186          return aStale == bStale ? 0 : (aStale ? 1 : -1);
187        }
188      }    
189        
190      /**
191       * Address matcher for matching an address to local address
192       */
193      static final AddressMatcher LOCAL_ADDRESS_MATCHER = new AddressMatcher() {
194        @Override
195        public boolean match(InetSocketAddress s) {
196          return NetUtils.isLocalAddress(s.getAddress());
197        };
198      };
199      
200      /**
201       * Whether the pathname is valid.  Currently prohibits relative paths, 
202       * names which contain a ":" or "//", or other non-canonical paths.
203       */
204      public static boolean isValidName(String src) {
205        // Path must be absolute.
206        if (!src.startsWith(Path.SEPARATOR)) {
207          return false;
208        }
209          
210        // Check for ".." "." ":" "/"
211        String[] components = StringUtils.split(src, '/');
212        for (int i = 0; i < components.length; i++) {
213          String element = components[i];
214          if (element.equals(".")  ||
215              (element.indexOf(":") >= 0)  ||
216              (element.indexOf("/") >= 0)) {
217            return false;
218          }
219          // ".." is allowed in path starting with /.reserved/.inodes
220          if (element.equals("..")) {
221            if (components.length > 4
222                && components[1].equals(FSDirectory.DOT_RESERVED_STRING)
223                && components[2].equals(FSDirectory.DOT_INODES_STRING)) {
224              continue;
225            }
226            return false;
227          }
228          // The string may start or end with a /, but not have
229          // "//" in the middle.
230          if (element.isEmpty() && i != components.length - 1 &&
231              i != 0) {
232            return false;
233          }
234        }
235        return true;
236      }
237    
238      /**
239       * Converts a byte array to a string using UTF8 encoding.
240       */
241      public static String bytes2String(byte[] bytes) {
242        return bytes2String(bytes, 0, bytes.length);
243      }
244      
245      /**
246       * Decode a specific range of bytes of the given byte array to a string
247       * using UTF8.
248       * 
249       * @param bytes The bytes to be decoded into characters
250       * @param offset The index of the first byte to decode
251       * @param length The number of bytes to decode
252       * @return The decoded string
253       */
254      public static String bytes2String(byte[] bytes, int offset, int length) {
255        try {
256          return new String(bytes, offset, length, "UTF8");
257        } catch(UnsupportedEncodingException e) {
258          assert false : "UTF8 encoding is not supported ";
259        }
260        return null;
261      }
262    
263      /**
264       * Converts a string to a byte array using UTF8 encoding.
265       */
266      public static byte[] string2Bytes(String str) {
267        return str.getBytes(Charsets.UTF_8);
268      }
269    
270      /**
271       * Given a list of path components returns a path as a UTF8 String
272       */
273      public static String byteArray2PathString(byte[][] pathComponents) {
274        if (pathComponents.length == 0) {
275          return "";
276        } else if (pathComponents.length == 1
277            && (pathComponents[0] == null || pathComponents[0].length == 0)) {
278          return Path.SEPARATOR;
279        }
280        StringBuilder result = new StringBuilder();
281        for (int i = 0; i < pathComponents.length; i++) {
282          result.append(new String(pathComponents[i], Charsets.UTF_8));
283          if (i < pathComponents.length - 1) {
284            result.append(Path.SEPARATOR_CHAR);
285          }
286        }
287        return result.toString();
288      }
289      
290      /**
291       * Given a list of path components returns a byte array
292       */
293      public static byte[] byteArray2bytes(byte[][] pathComponents) {
294        if (pathComponents.length == 0) {
295          return EMPTY_BYTES;
296        } else if (pathComponents.length == 1
297            && (pathComponents[0] == null || pathComponents[0].length == 0)) {
298          return new byte[]{(byte) Path.SEPARATOR_CHAR};
299        }
300        int length = 0;
301        for (int i = 0; i < pathComponents.length; i++) {
302          length += pathComponents[i].length;
303          if (i < pathComponents.length - 1) {
304            length++; // for SEPARATOR
305          }
306        }
307        byte[] path = new byte[length];
308        int index = 0;
309        for (int i = 0; i < pathComponents.length; i++) {
310          System.arraycopy(pathComponents[i], 0, path, index,
311              pathComponents[i].length);
312          index += pathComponents[i].length;
313          if (i < pathComponents.length - 1) {
314            path[index] = (byte) Path.SEPARATOR_CHAR;
315            index++;
316          }
317        }
318        return path;
319      }
320    
321      /** Convert an object representing a path to a string. */
322      public static String path2String(final Object path) {
323        return path == null? null
324            : path instanceof String? (String)path
325            : path instanceof byte[][]? byteArray2PathString((byte[][])path)
326            : path.toString();
327      }
328    
329      /**
330       * Splits the array of bytes into array of arrays of bytes
331       * on byte separator
332       * @param bytes the array of bytes to split
333       * @param separator the delimiting byte
334       */
335      public static byte[][] bytes2byteArray(byte[] bytes, byte separator) {
336        return bytes2byteArray(bytes, bytes.length, separator);
337      }
338    
339      /**
340       * Splits first len bytes in bytes to array of arrays of bytes
341       * on byte separator
342       * @param bytes the byte array to split
343       * @param len the number of bytes to split
344       * @param separator the delimiting byte
345       */
346      public static byte[][] bytes2byteArray(byte[] bytes,
347                                             int len,
348                                             byte separator) {
349        assert len <= bytes.length;
350        int splits = 0;
351        if (len == 0) {
352          return new byte[][]{null};
353        }
354        // Count the splits. Omit multiple separators and the last one
355        for (int i = 0; i < len; i++) {
356          if (bytes[i] == separator) {
357            splits++;
358          }
359        }
360        int last = len - 1;
361        while (last > -1 && bytes[last--] == separator) {
362          splits--;
363        }
364        if (splits == 0 && bytes[0] == separator) {
365          return new byte[][]{null};
366        }
367        splits++;
368        byte[][] result = new byte[splits][];
369        int startIndex = 0;
370        int nextIndex = 0;
371        int index = 0;
372        // Build the splits
373        while (index < splits) {
374          while (nextIndex < len && bytes[nextIndex] != separator) {
375            nextIndex++;
376          }
377          result[index] = new byte[nextIndex - startIndex];
378          System.arraycopy(bytes, startIndex, result[index], 0, nextIndex
379                  - startIndex);
380          index++;
381          startIndex = nextIndex + 1;
382          nextIndex = startIndex;
383        }
384        return result;
385      }
386      
387      /**
388       * Convert a LocatedBlocks to BlockLocations[]
389       * @param blocks a LocatedBlocks
390       * @return an array of BlockLocations
391       */
392      public static BlockLocation[] locatedBlocks2Locations(LocatedBlocks blocks) {
393        if (blocks == null) {
394          return new BlockLocation[0];
395        }
396        return locatedBlocks2Locations(blocks.getLocatedBlocks());
397      }
398      
399      /**
400       * Convert a List<LocatedBlock> to BlockLocation[]
401       * @param blocks A List<LocatedBlock> to be converted
402       * @return converted array of BlockLocation
403       */
404      public static BlockLocation[] locatedBlocks2Locations(List<LocatedBlock> blocks) {
405        if (blocks == null) {
406          return new BlockLocation[0];
407        }
408        int nrBlocks = blocks.size();
409        BlockLocation[] blkLocations = new BlockLocation[nrBlocks];
410        if (nrBlocks == 0) {
411          return blkLocations;
412        }
413        int idx = 0;
414        for (LocatedBlock blk : blocks) {
415          assert idx < nrBlocks : "Incorrect index";
416          DatanodeInfo[] locations = blk.getLocations();
417          String[] hosts = new String[locations.length];
418          String[] xferAddrs = new String[locations.length];
419          String[] racks = new String[locations.length];
420          for (int hCnt = 0; hCnt < locations.length; hCnt++) {
421            hosts[hCnt] = locations[hCnt].getHostName();
422            xferAddrs[hCnt] = locations[hCnt].getXferAddr();
423            NodeBase node = new NodeBase(xferAddrs[hCnt], 
424                                         locations[hCnt].getNetworkLocation());
425            racks[hCnt] = node.toString();
426          }
427          blkLocations[idx] = new BlockLocation(xferAddrs, hosts, racks,
428                                                blk.getStartOffset(),
429                                                blk.getBlockSize(),
430                                                blk.isCorrupt());
431          idx++;
432        }
433        return blkLocations;
434      }
435    
436      /**
437       * Returns collection of nameservice Ids from the configuration.
438       * @param conf configuration
439       * @return collection of nameservice Ids, or null if not specified
440       */
441      public static Collection<String> getNameServiceIds(Configuration conf) {
442        return conf.getTrimmedStringCollection(DFS_NAMESERVICES);
443      }
444    
445      /**
446       * @return <code>coll</code> if it is non-null and non-empty. Otherwise,
447       * returns a list with a single null value.
448       */
449      private static Collection<String> emptyAsSingletonNull(Collection<String> coll) {
450        if (coll == null || coll.isEmpty()) {
451          return Collections.singletonList(null);
452        } else {
453          return coll;
454        }
455      }
456      
457      /**
458       * Namenode HighAvailability related configuration.
459       * Returns collection of namenode Ids from the configuration. One logical id
460       * for each namenode in the in the HA setup.
461       * 
462       * @param conf configuration
463       * @param nsId the nameservice ID to look at, or null for non-federated 
464       * @return collection of namenode Ids
465       */
466      public static Collection<String> getNameNodeIds(Configuration conf, String nsId) {
467        String key = addSuffix(DFS_HA_NAMENODES_KEY_PREFIX, nsId);
468        return conf.getTrimmedStringCollection(key);
469      }
470      
471      /**
472       * Given a list of keys in the order of preference, returns a value
473       * for the key in the given order from the configuration.
474       * @param defaultValue default value to return, when key was not found
475       * @param keySuffix suffix to add to the key, if it is not null
476       * @param conf Configuration
477       * @param keys list of keys in the order of preference
478       * @return value of the key or default if a key was not found in configuration
479       */
480      private static String getConfValue(String defaultValue, String keySuffix,
481          Configuration conf, String... keys) {
482        String value = null;
483        for (String key : keys) {
484          key = addSuffix(key, keySuffix);
485          value = conf.get(key);
486          if (value != null) {
487            break;
488          }
489        }
490        if (value == null) {
491          value = defaultValue;
492        }
493        return value;
494      }
495      
496      /** Add non empty and non null suffix to a key */
497      private static String addSuffix(String key, String suffix) {
498        if (suffix == null || suffix.isEmpty()) {
499          return key;
500        }
501        assert !suffix.startsWith(".") :
502          "suffix '" + suffix + "' should not already have '.' prepended.";
503        return key + "." + suffix;
504      }
505      
506      /** Concatenate list of suffix strings '.' separated */
507      private static String concatSuffixes(String... suffixes) {
508        if (suffixes == null) {
509          return null;
510        }
511        return Joiner.on(".").skipNulls().join(suffixes);
512      }
513      
514      /**
515       * Return configuration key of format key.suffix1.suffix2...suffixN
516       */
517      public static String addKeySuffixes(String key, String... suffixes) {
518        String keySuffix = concatSuffixes(suffixes);
519        return addSuffix(key, keySuffix);
520      }
521      
522      /**
523       * Returns the configured address for all NameNodes in the cluster.
524       * @param conf configuration
525       * @param defaultAddress default address to return in case key is not found.
526       * @param keys Set of keys to look for in the order of preference
527       * @return a map(nameserviceId to map(namenodeId to InetSocketAddress))
528       */
529      private static Map<String, Map<String, InetSocketAddress>>
530        getAddresses(Configuration conf,
531          String defaultAddress, String... keys) {
532        Collection<String> nameserviceIds = getNameServiceIds(conf);
533        
534        // Look for configurations of the form <key>[.<nameserviceId>][.<namenodeId>]
535        // across all of the configured nameservices and namenodes.
536        Map<String, Map<String, InetSocketAddress>> ret = Maps.newLinkedHashMap();
537        for (String nsId : emptyAsSingletonNull(nameserviceIds)) {
538          Map<String, InetSocketAddress> isas =
539            getAddressesForNameserviceId(conf, nsId, defaultAddress, keys);
540          if (!isas.isEmpty()) {
541            ret.put(nsId, isas);
542          }
543        }
544        return ret;
545      }
546    
547      private static Map<String, InetSocketAddress> getAddressesForNameserviceId(
548          Configuration conf, String nsId, String defaultValue,
549          String[] keys) {
550        Collection<String> nnIds = getNameNodeIds(conf, nsId);
551        Map<String, InetSocketAddress> ret = Maps.newHashMap();
552        for (String nnId : emptyAsSingletonNull(nnIds)) {
553          String suffix = concatSuffixes(nsId, nnId);
554          String address = getConfValue(defaultValue, suffix, conf, keys);
555          if (address != null) {
556            InetSocketAddress isa = NetUtils.createSocketAddr(address);
557            ret.put(nnId, isa);
558          }
559        }
560        return ret;
561      }
562    
563      /**
564       * @return a collection of all configured NN Kerberos principals.
565       */
566      public static Set<String> getAllNnPrincipals(Configuration conf) throws IOException {
567        Set<String> principals = new HashSet<String>();
568        for (String nsId : DFSUtil.getNameServiceIds(conf)) {
569          if (HAUtil.isHAEnabled(conf, nsId)) {
570            for (String nnId : DFSUtil.getNameNodeIds(conf, nsId)) {
571              Configuration confForNn = new Configuration(conf);
572              NameNode.initializeGenericKeys(confForNn, nsId, nnId);
573              String principal = SecurityUtil.getServerPrincipal(confForNn
574                  .get(DFSConfigKeys.DFS_NAMENODE_USER_NAME_KEY),
575                  NameNode.getAddress(confForNn).getHostName());
576              principals.add(principal);
577            }
578          } else {
579            Configuration confForNn = new Configuration(conf);
580            NameNode.initializeGenericKeys(confForNn, nsId, null);
581            String principal = SecurityUtil.getServerPrincipal(confForNn
582                .get(DFSConfigKeys.DFS_NAMENODE_USER_NAME_KEY),
583                NameNode.getAddress(confForNn).getHostName());
584            principals.add(principal);
585          }
586        }
587    
588        return principals;
589      }
590    
591      /**
592       * Returns list of InetSocketAddress corresponding to HA NN RPC addresses from
593       * the configuration.
594       * 
595       * @param conf configuration
596       * @return list of InetSocketAddresses
597       */
598      public static Map<String, Map<String, InetSocketAddress>> getHaNnRpcAddresses(
599          Configuration conf) {
600        return getAddresses(conf, null, DFSConfigKeys.DFS_NAMENODE_RPC_ADDRESS_KEY);
601      }
602      
603      /**
604       * Returns list of InetSocketAddress corresponding to  backup node rpc 
605       * addresses from the configuration.
606       * 
607       * @param conf configuration
608       * @return list of InetSocketAddresses
609       * @throws IOException on error
610       */
611      public static Map<String, Map<String, InetSocketAddress>> getBackupNodeAddresses(
612          Configuration conf) throws IOException {
613        Map<String, Map<String, InetSocketAddress>> addressList = getAddresses(conf,
614            null, DFS_NAMENODE_BACKUP_ADDRESS_KEY);
615        if (addressList.isEmpty()) {
616          throw new IOException("Incorrect configuration: backup node address "
617              + DFS_NAMENODE_BACKUP_ADDRESS_KEY + " is not configured.");
618        }
619        return addressList;
620      }
621    
622      /**
623       * Returns list of InetSocketAddresses of corresponding to secondary namenode
624       * http addresses from the configuration.
625       * 
626       * @param conf configuration
627       * @return list of InetSocketAddresses
628       * @throws IOException on error
629       */
630      public static Map<String, Map<String, InetSocketAddress>> getSecondaryNameNodeAddresses(
631          Configuration conf) throws IOException {
632        Map<String, Map<String, InetSocketAddress>> addressList = getAddresses(conf, null,
633            DFS_NAMENODE_SECONDARY_HTTP_ADDRESS_KEY);
634        if (addressList.isEmpty()) {
635          throw new IOException("Incorrect configuration: secondary namenode address "
636              + DFS_NAMENODE_SECONDARY_HTTP_ADDRESS_KEY + " is not configured.");
637        }
638        return addressList;
639      }
640    
641      /**
642       * Returns list of InetSocketAddresses corresponding to namenodes from the
643       * configuration. Note this is to be used by datanodes to get the list of
644       * namenode addresses to talk to.
645       * 
646       * Returns namenode address specifically configured for datanodes (using
647       * service ports), if found. If not, regular RPC address configured for other
648       * clients is returned.
649       * 
650       * @param conf configuration
651       * @return list of InetSocketAddress
652       * @throws IOException on error
653       */
654      public static Map<String, Map<String, InetSocketAddress>> getNNServiceRpcAddresses(
655          Configuration conf) throws IOException {
656        // Use default address as fall back
657        String defaultAddress;
658        try {
659          defaultAddress = NetUtils.getHostPortString(NameNode.getAddress(conf));
660        } catch (IllegalArgumentException e) {
661          defaultAddress = null;
662        }
663        
664        Map<String, Map<String, InetSocketAddress>> addressList =
665          getAddresses(conf, defaultAddress,
666            DFS_NAMENODE_SERVICE_RPC_ADDRESS_KEY, DFS_NAMENODE_RPC_ADDRESS_KEY);
667        if (addressList.isEmpty()) {
668          throw new IOException("Incorrect configuration: namenode address "
669              + DFS_NAMENODE_SERVICE_RPC_ADDRESS_KEY + " or "  
670              + DFS_NAMENODE_RPC_ADDRESS_KEY
671              + " is not configured.");
672        }
673        return addressList;
674      }
675      
676      /**
677       * Flatten the given map, as returned by other functions in this class,
678       * into a flat list of {@link ConfiguredNNAddress} instances.
679       */
680      public static List<ConfiguredNNAddress> flattenAddressMap(
681          Map<String, Map<String, InetSocketAddress>> map) {
682        List<ConfiguredNNAddress> ret = Lists.newArrayList();
683        
684        for (Map.Entry<String, Map<String, InetSocketAddress>> entry :
685          map.entrySet()) {
686          String nsId = entry.getKey();
687          Map<String, InetSocketAddress> nnMap = entry.getValue();
688          for (Map.Entry<String, InetSocketAddress> e2 : nnMap.entrySet()) {
689            String nnId = e2.getKey();
690            InetSocketAddress addr = e2.getValue();
691            
692            ret.add(new ConfiguredNNAddress(nsId, nnId, addr));
693          }
694        }
695        return ret;
696      }
697    
698      /**
699       * Format the given map, as returned by other functions in this class,
700       * into a string suitable for debugging display. The format of this string
701       * should not be considered an interface, and is liable to change.
702       */
703      public static String addressMapToString(
704          Map<String, Map<String, InetSocketAddress>> map) {
705        StringBuilder b = new StringBuilder();
706        for (Map.Entry<String, Map<String, InetSocketAddress>> entry :
707             map.entrySet()) {
708          String nsId = entry.getKey();
709          Map<String, InetSocketAddress> nnMap = entry.getValue();
710          b.append("Nameservice <").append(nsId).append(">:").append("\n");
711          for (Map.Entry<String, InetSocketAddress> e2 : nnMap.entrySet()) {
712            b.append("  NN ID ").append(e2.getKey())
713              .append(" => ").append(e2.getValue()).append("\n");
714          }
715        }
716        return b.toString();
717      }
718      
719      public static String nnAddressesAsString(Configuration conf) {
720        Map<String, Map<String, InetSocketAddress>> addresses =
721          getHaNnRpcAddresses(conf);
722        return addressMapToString(addresses);
723      }
724    
725      /**
726       * Represent one of the NameNodes configured in the cluster.
727       */
728      public static class ConfiguredNNAddress {
729        private final String nameserviceId;
730        private final String namenodeId;
731        private final InetSocketAddress addr;
732    
733        private ConfiguredNNAddress(String nameserviceId, String namenodeId,
734            InetSocketAddress addr) {
735          this.nameserviceId = nameserviceId;
736          this.namenodeId = namenodeId;
737          this.addr = addr;
738        }
739    
740        public String getNameserviceId() {
741          return nameserviceId;
742        }
743    
744        public String getNamenodeId() {
745          return namenodeId;
746        }
747    
748        public InetSocketAddress getAddress() {
749          return addr;
750        }
751        
752        @Override
753        public String toString() {
754          return "ConfiguredNNAddress[nsId=" + nameserviceId + ";" +
755            "nnId=" + namenodeId + ";addr=" + addr + "]";
756        }
757      }
758      
759      /**
760       * Get a URI for each configured nameservice. If a nameservice is
761       * HA-enabled, then the logical URI of the nameservice is returned. If the
762       * nameservice is not HA-enabled, then a URI corresponding to an RPC address
763       * of the single NN for that nameservice is returned, preferring the service
764       * RPC address over the client RPC address.
765       * 
766       * @param conf configuration
767       * @return a collection of all configured NN URIs, preferring service
768       *         addresses
769       */
770      public static Collection<URI> getNsServiceRpcUris(Configuration conf) {
771        return getNameServiceUris(conf,
772            DFSConfigKeys.DFS_NAMENODE_SERVICE_RPC_ADDRESS_KEY,
773            DFSConfigKeys.DFS_NAMENODE_RPC_ADDRESS_KEY);
774      }
775    
776      /**
777       * Get a URI for each configured nameservice. If a nameservice is
778       * HA-enabled, then the logical URI of the nameservice is returned. If the
779       * nameservice is not HA-enabled, then a URI corresponding to the address of
780       * the single NN for that nameservice is returned.
781       * 
782       * @param conf configuration
783       * @param keys configuration keys to try in order to get the URI for non-HA
784       *        nameservices
785       * @return a collection of all configured NN URIs
786       */
787      public static Collection<URI> getNameServiceUris(Configuration conf,
788          String... keys) {
789        Set<URI> ret = new HashSet<URI>();
790        
791        // We're passed multiple possible configuration keys for any given NN or HA
792        // nameservice, and search the config in order of these keys. In order to
793        // make sure that a later config lookup (e.g. fs.defaultFS) doesn't add a
794        // URI for a config key for which we've already found a preferred entry, we
795        // keep track of non-preferred keys here.
796        Set<URI> nonPreferredUris = new HashSet<URI>();
797        
798        for (String nsId : getNameServiceIds(conf)) {
799          if (HAUtil.isHAEnabled(conf, nsId)) {
800            // Add the logical URI of the nameservice.
801            try {
802              ret.add(new URI(HdfsConstants.HDFS_URI_SCHEME + "://" + nsId));
803            } catch (URISyntaxException ue) {
804              throw new IllegalArgumentException(ue);
805            }
806          } else {
807            // Add the URI corresponding to the address of the NN.
808            boolean uriFound = false;
809            for (String key : keys) {
810              String addr = conf.get(concatSuffixes(key, nsId));
811              if (addr != null) {
812                URI uri = createUri(HdfsConstants.HDFS_URI_SCHEME,
813                    NetUtils.createSocketAddr(addr));
814                if (!uriFound) {
815                  uriFound = true;
816                  ret.add(uri);
817                } else {
818                  nonPreferredUris.add(uri);
819                }
820              }
821            }
822          }
823        }
824        
825        // Add the generic configuration keys.
826        boolean uriFound = false;
827        for (String key : keys) {
828          String addr = conf.get(key);
829          if (addr != null) {
830            URI uri = createUri("hdfs", NetUtils.createSocketAddr(addr));
831            if (!uriFound) {
832              uriFound = true;
833              ret.add(uri);
834            } else {
835              nonPreferredUris.add(uri);
836            }
837          }
838        }
839        
840        // Add the default URI if it is an HDFS URI.
841        URI defaultUri = FileSystem.getDefaultUri(conf);
842        // checks if defaultUri is ip:port format
843        // and convert it to hostname:port format
844        if (defaultUri != null && (defaultUri.getPort() != -1)) {
845          defaultUri = createUri(defaultUri.getScheme(),
846              NetUtils.createSocketAddr(defaultUri.getHost(), 
847                  defaultUri.getPort()));
848        }
849        if (defaultUri != null &&
850            HdfsConstants.HDFS_URI_SCHEME.equals(defaultUri.getScheme()) &&
851            !nonPreferredUris.contains(defaultUri)) {
852          ret.add(defaultUri);
853        }
854        
855        return ret;
856      }
857    
858      /**
859       * Given the InetSocketAddress this method returns the nameservice Id
860       * corresponding to the key with matching address, by doing a reverse 
861       * lookup on the list of nameservices until it finds a match.
862       * 
863       * Since the process of resolving URIs to Addresses is slightly expensive,
864       * this utility method should not be used in performance-critical routines.
865       * 
866       * @param conf - configuration
867       * @param address - InetSocketAddress for configured communication with NN.
868       *     Configured addresses are typically given as URIs, but we may have to
869       *     compare against a URI typed in by a human, or the server name may be
870       *     aliased, so we compare unambiguous InetSocketAddresses instead of just
871       *     comparing URI substrings.
872       * @param keys - list of configured communication parameters that should
873       *     be checked for matches.  For example, to compare against RPC addresses,
874       *     provide the list DFS_NAMENODE_SERVICE_RPC_ADDRESS_KEY,
875       *     DFS_NAMENODE_RPC_ADDRESS_KEY.  Use the generic parameter keys,
876       *     not the NameServiceId-suffixed keys.
877       * @return nameserviceId, or null if no match found
878       */
879      public static String getNameServiceIdFromAddress(final Configuration conf, 
880          final InetSocketAddress address, String... keys) {
881        // Configuration with a single namenode and no nameserviceId
882        String[] ids = getSuffixIDs(conf, address, keys);
883        return (ids != null) ? ids[0] : null;
884      }
885      
886      /**
887       * return server http or https address from the configuration for a
888       * given namenode rpc address.
889       * @param conf
890       * @param namenodeAddr - namenode RPC address
891       * @param httpsAddress -If true, and if security is enabled, returns server 
892       *                      https address. If false, returns server http address.
893       * @return server http or https address
894       * @throws IOException 
895       */
896      public static String getInfoServer(InetSocketAddress namenodeAddr,
897          Configuration conf, boolean httpsAddress) throws IOException {
898        boolean securityOn = UserGroupInformation.isSecurityEnabled();
899        String httpAddressKey = (securityOn && httpsAddress) ? 
900            DFS_NAMENODE_HTTPS_ADDRESS_KEY : DFS_NAMENODE_HTTP_ADDRESS_KEY;
901        String httpAddressDefault = (securityOn && httpsAddress) ? 
902            DFS_NAMENODE_HTTPS_ADDRESS_DEFAULT : DFS_NAMENODE_HTTP_ADDRESS_DEFAULT;
903          
904        String suffixes[];
905        if (namenodeAddr != null) {
906          // if non-default namenode, try reverse look up 
907          // the nameServiceID if it is available
908          suffixes = getSuffixIDs(conf, namenodeAddr,
909              DFSConfigKeys.DFS_NAMENODE_SERVICE_RPC_ADDRESS_KEY,
910              DFSConfigKeys.DFS_NAMENODE_RPC_ADDRESS_KEY);
911        } else {
912          suffixes = new String[2];
913        }
914        String configuredInfoAddr = getSuffixedConf(conf, httpAddressKey,
915            httpAddressDefault, suffixes);
916        if (namenodeAddr != null) {
917          return substituteForWildcardAddress(configuredInfoAddr,
918              namenodeAddr.getHostName());
919        } else {
920          return configuredInfoAddr;
921        }
922      }
923      
924    
925      /**
926       * Substitute a default host in the case that an address has been configured
927       * with a wildcard. This is used, for example, when determining the HTTP
928       * address of the NN -- if it's configured to bind to 0.0.0.0, we want to
929       * substitute the hostname from the filesystem URI rather than trying to
930       * connect to 0.0.0.0.
931       * @param configuredAddress the address found in the configuration
932       * @param defaultHost the host to substitute with, if configuredAddress
933       * is a local/wildcard address.
934       * @return the substituted address
935       * @throws IOException if it is a wildcard address and security is enabled
936       */
937      public static String substituteForWildcardAddress(String configuredAddress,
938          String defaultHost) throws IOException {
939        InetSocketAddress sockAddr = NetUtils.createSocketAddr(configuredAddress);
940        InetSocketAddress defaultSockAddr = NetUtils.createSocketAddr(defaultHost
941            + ":0");
942        if (sockAddr.getAddress().isAnyLocalAddress()) {
943          if (UserGroupInformation.isSecurityEnabled() &&
944              defaultSockAddr.getAddress().isAnyLocalAddress()) {
945            throw new IOException("Cannot use a wildcard address with security. " +
946                "Must explicitly set bind address for Kerberos");
947          }
948          return defaultHost + ":" + sockAddr.getPort();
949        } else {
950          return configuredAddress;
951        }
952      }
953      
954      private static String getSuffixedConf(Configuration conf,
955          String key, String defaultVal, String[] suffixes) {
956        String ret = conf.get(DFSUtil.addKeySuffixes(key, suffixes));
957        if (ret != null) {
958          return ret;
959        }
960        return conf.get(key, defaultVal);
961      }
962      
963      /**
964       * Sets the node specific setting into generic configuration key. Looks up
965       * value of "key.nameserviceId.namenodeId" and if found sets that value into 
966       * generic key in the conf. If this is not found, falls back to
967       * "key.nameserviceId" and then the unmodified key.
968       *
969       * Note that this only modifies the runtime conf.
970       * 
971       * @param conf
972       *          Configuration object to lookup specific key and to set the value
973       *          to the key passed. Note the conf object is modified.
974       * @param nameserviceId
975       *          nameservice Id to construct the node specific key. Pass null if
976       *          federation is not configuration.
977       * @param nnId
978       *          namenode Id to construct the node specific key. Pass null if
979       *          HA is not configured.
980       * @param keys
981       *          The key for which node specific value is looked up
982       */
983      public static void setGenericConf(Configuration conf,
984          String nameserviceId, String nnId, String... keys) {
985        for (String key : keys) {
986          String value = conf.get(addKeySuffixes(key, nameserviceId, nnId));
987          if (value != null) {
988            conf.set(key, value);
989            continue;
990          }
991          value = conf.get(addKeySuffixes(key, nameserviceId));
992          if (value != null) {
993            conf.set(key, value);
994          }
995        }
996      }
997      
998      /** Return used as percentage of capacity */
999      public static float getPercentUsed(long used, long capacity) {
1000        return capacity <= 0 ? 100 : (used * 100.0f)/capacity; 
1001      }
1002      
1003      /** Return remaining as percentage of capacity */
1004      public static float getPercentRemaining(long remaining, long capacity) {
1005        return capacity <= 0 ? 0 : (remaining * 100.0f)/capacity; 
1006      }
1007    
1008      /** Convert percentage to a string. */
1009      public static String percent2String(double percentage) {
1010        return StringUtils.format("%.2f%%", percentage);
1011      }
1012    
1013      /**
1014       * Round bytes to GiB (gibibyte)
1015       * @param bytes number of bytes
1016       * @return number of GiB
1017       */
1018      public static int roundBytesToGB(long bytes) {
1019        return Math.round((float)bytes/ 1024 / 1024 / 1024);
1020      }
1021      
1022      /** Create a {@link ClientDatanodeProtocol} proxy */
1023      public static ClientDatanodeProtocol createClientDatanodeProtocolProxy(
1024          DatanodeID datanodeid, Configuration conf, int socketTimeout,
1025          boolean connectToDnViaHostname, LocatedBlock locatedBlock) throws IOException {
1026        return new ClientDatanodeProtocolTranslatorPB(datanodeid, conf, socketTimeout,
1027            connectToDnViaHostname, locatedBlock);
1028      }
1029      
1030      /** Create {@link ClientDatanodeProtocol} proxy using kerberos ticket */
1031      static ClientDatanodeProtocol createClientDatanodeProtocolProxy(
1032          DatanodeID datanodeid, Configuration conf, int socketTimeout,
1033          boolean connectToDnViaHostname) throws IOException {
1034        return new ClientDatanodeProtocolTranslatorPB(
1035            datanodeid, conf, socketTimeout, connectToDnViaHostname);
1036      }
1037      
1038      /** Create a {@link ClientDatanodeProtocol} proxy */
1039      public static ClientDatanodeProtocol createClientDatanodeProtocolProxy(
1040          InetSocketAddress addr, UserGroupInformation ticket, Configuration conf,
1041          SocketFactory factory) throws IOException {
1042        return new ClientDatanodeProtocolTranslatorPB(addr, ticket, conf, factory);
1043      }
1044    
1045      /**
1046       * Get nameservice Id for the {@link NameNode} based on namenode RPC address
1047       * matching the local node address.
1048       */
1049      public static String getNamenodeNameServiceId(Configuration conf) {
1050        return getNameServiceId(conf, DFS_NAMENODE_RPC_ADDRESS_KEY);
1051      }
1052      
1053      /**
1054       * Get nameservice Id for the BackupNode based on backup node RPC address
1055       * matching the local node address.
1056       */
1057      public static String getBackupNameServiceId(Configuration conf) {
1058        return getNameServiceId(conf, DFS_NAMENODE_BACKUP_ADDRESS_KEY);
1059      }
1060      
1061      /**
1062       * Get nameservice Id for the secondary node based on secondary http address
1063       * matching the local node address.
1064       */
1065      public static String getSecondaryNameServiceId(Configuration conf) {
1066        return getNameServiceId(conf, DFS_NAMENODE_SECONDARY_HTTP_ADDRESS_KEY);
1067      }
1068      
1069      /**
1070       * Get the nameservice Id by matching the {@code addressKey} with the
1071       * the address of the local node. 
1072       * 
1073       * If {@link DFSConfigKeys#DFS_NAMESERVICE_ID} is not specifically
1074       * configured, and more than one nameservice Id is configured, this method 
1075       * determines the nameservice Id by matching the local node's address with the
1076       * configured addresses. When a match is found, it returns the nameservice Id
1077       * from the corresponding configuration key.
1078       * 
1079       * @param conf Configuration
1080       * @param addressKey configuration key to get the address.
1081       * @return nameservice Id on success, null if federation is not configured.
1082       * @throws HadoopIllegalArgumentException on error
1083       */
1084      private static String getNameServiceId(Configuration conf, String addressKey) {
1085        String nameserviceId = conf.get(DFS_NAMESERVICE_ID);
1086        if (nameserviceId != null) {
1087          return nameserviceId;
1088        }
1089        Collection<String> nsIds = getNameServiceIds(conf);
1090        if (1 == nsIds.size()) {
1091          return nsIds.toArray(new String[1])[0];
1092        }
1093        String nnId = conf.get(DFS_HA_NAMENODE_ID_KEY);
1094        
1095        return getSuffixIDs(conf, addressKey, null, nnId, LOCAL_ADDRESS_MATCHER)[0];
1096      }
1097      
1098      /**
1099       * Returns nameservice Id and namenode Id when the local host matches the
1100       * configuration parameter {@code addressKey}.<nameservice Id>.<namenode Id>
1101       * 
1102       * @param conf Configuration
1103       * @param addressKey configuration key corresponding to the address.
1104       * @param knownNsId only look at configs for the given nameservice, if not-null
1105       * @param knownNNId only look at configs for the given namenode, if not null
1106       * @param matcher matching criteria for matching the address
1107       * @return Array with nameservice Id and namenode Id on success. First element
1108       *         in the array is nameservice Id and second element is namenode Id.
1109       *         Null value indicates that the configuration does not have the the
1110       *         Id.
1111       * @throws HadoopIllegalArgumentException on error
1112       */
1113      static String[] getSuffixIDs(final Configuration conf, final String addressKey,
1114          String knownNsId, String knownNNId,
1115          final AddressMatcher matcher) {
1116        String nameserviceId = null;
1117        String namenodeId = null;
1118        int found = 0;
1119        
1120        Collection<String> nsIds = getNameServiceIds(conf);
1121        for (String nsId : emptyAsSingletonNull(nsIds)) {
1122          if (knownNsId != null && !knownNsId.equals(nsId)) {
1123            continue;
1124          }
1125          
1126          Collection<String> nnIds = getNameNodeIds(conf, nsId);
1127          for (String nnId : emptyAsSingletonNull(nnIds)) {
1128            if (LOG.isTraceEnabled()) {
1129              LOG.trace(String.format("addressKey: %s nsId: %s nnId: %s",
1130                  addressKey, nsId, nnId));
1131            }
1132            if (knownNNId != null && !knownNNId.equals(nnId)) {
1133              continue;
1134            }
1135            String key = addKeySuffixes(addressKey, nsId, nnId);
1136            String addr = conf.get(key);
1137            if (addr == null) {
1138              continue;
1139            }
1140            InetSocketAddress s = null;
1141            try {
1142              s = NetUtils.createSocketAddr(addr);
1143            } catch (Exception e) {
1144              LOG.warn("Exception in creating socket address " + addr, e);
1145              continue;
1146            }
1147            if (!s.isUnresolved() && matcher.match(s)) {
1148              nameserviceId = nsId;
1149              namenodeId = nnId;
1150              found++;
1151            }
1152          }
1153        }
1154        if (found > 1) { // Only one address must match the local address
1155          String msg = "Configuration has multiple addresses that match "
1156              + "local node's address. Please configure the system with "
1157              + DFS_NAMESERVICE_ID + " and "
1158              + DFS_HA_NAMENODE_ID_KEY;
1159          throw new HadoopIllegalArgumentException(msg);
1160        }
1161        return new String[] { nameserviceId, namenodeId };
1162      }
1163      
1164      /**
1165       * For given set of {@code keys} adds nameservice Id and or namenode Id
1166       * and returns {nameserviceId, namenodeId} when address match is found.
1167       * @see #getSuffixIDs(Configuration, String, AddressMatcher)
1168       */
1169      static String[] getSuffixIDs(final Configuration conf,
1170          final InetSocketAddress address, final String... keys) {
1171        AddressMatcher matcher = new AddressMatcher() {
1172         @Override
1173          public boolean match(InetSocketAddress s) {
1174            return address.equals(s);
1175          } 
1176        };
1177        
1178        for (String key : keys) {
1179          String[] ids = getSuffixIDs(conf, key, null, null, matcher);
1180          if (ids != null && (ids [0] != null || ids[1] != null)) {
1181            return ids;
1182          }
1183        }
1184        return null;
1185      }
1186      
1187      private interface AddressMatcher {
1188        public boolean match(InetSocketAddress s);
1189      }
1190    
1191      /** Create a URI from the scheme and address */
1192      public static URI createUri(String scheme, InetSocketAddress address) {
1193        try {
1194          return new URI(scheme, null, address.getHostName(), address.getPort(),
1195              null, null, null);
1196        } catch (URISyntaxException ue) {
1197          throw new IllegalArgumentException(ue);
1198        }
1199      }
1200      
1201      /**
1202       * Add protobuf based protocol to the {@link org.apache.hadoop.ipc.RPC.Server}
1203       * @param conf configuration
1204       * @param protocol Protocol interface
1205       * @param service service that implements the protocol
1206       * @param server RPC server to which the protocol & implementation is added to
1207       * @throws IOException
1208       */
1209      public static void addPBProtocol(Configuration conf, Class<?> protocol,
1210          BlockingService service, RPC.Server server) throws IOException {
1211        RPC.setProtocolEngine(conf, protocol, ProtobufRpcEngine.class);
1212        server.addProtocol(RPC.RpcKind.RPC_PROTOCOL_BUFFER, protocol, service);
1213      }
1214    
1215      /**
1216       * Map a logical namenode ID to its service address. Use the given
1217       * nameservice if specified, or the configured one if none is given.
1218       *
1219       * @param conf Configuration
1220       * @param nsId which nameservice nnId is a part of, optional
1221       * @param nnId the namenode ID to get the service addr for
1222       * @return the service addr, null if it could not be determined
1223       */
1224      public static String getNamenodeServiceAddr(final Configuration conf,
1225          String nsId, String nnId) {
1226    
1227        if (nsId == null) {
1228          nsId = getOnlyNameServiceIdOrNull(conf);
1229        }
1230    
1231        String serviceAddrKey = concatSuffixes(
1232            DFSConfigKeys.DFS_NAMENODE_SERVICE_RPC_ADDRESS_KEY, nsId, nnId);
1233    
1234        String addrKey = concatSuffixes(
1235            DFSConfigKeys.DFS_NAMENODE_RPC_ADDRESS_KEY, nsId, nnId);
1236    
1237        String serviceRpcAddr = conf.get(serviceAddrKey);
1238        if (serviceRpcAddr == null) {
1239          serviceRpcAddr = conf.get(addrKey);
1240        }
1241        return serviceRpcAddr;
1242      }
1243    
1244      /**
1245       * If the configuration refers to only a single nameservice, return the
1246       * name of that nameservice. If it refers to 0 or more than 1, return null.
1247       */
1248      public static String getOnlyNameServiceIdOrNull(Configuration conf) {
1249        Collection<String> nsIds = getNameServiceIds(conf);
1250        if (1 == nsIds.size()) {
1251          return nsIds.toArray(new String[1])[0];
1252        } else {
1253          // No nameservice ID was given and more than one is configured
1254          return null;
1255        }
1256      }
1257      
1258      public static Options helpOptions = new Options();
1259      public static Option helpOpt = new Option("h", "help", false,
1260          "get help information");
1261    
1262      static {
1263        helpOptions.addOption(helpOpt);
1264      }
1265    
1266      /**
1267       * Parse the arguments for commands
1268       * 
1269       * @param args the argument to be parsed
1270       * @param helpDescription help information to be printed out
1271       * @param out Printer
1272       * @param printGenericCommandUsage whether to print the 
1273       *              generic command usage defined in ToolRunner
1274       * @return true when the argument matches help option, false if not
1275       */
1276      public static boolean parseHelpArgument(String[] args,
1277          String helpDescription, PrintStream out, boolean printGenericCommandUsage) {
1278        if (args.length == 1) {
1279          try {
1280            CommandLineParser parser = new PosixParser();
1281            CommandLine cmdLine = parser.parse(helpOptions, args);
1282            if (cmdLine.hasOption(helpOpt.getOpt())
1283                || cmdLine.hasOption(helpOpt.getLongOpt())) {
1284              // should print out the help information
1285              out.println(helpDescription + "\n");
1286              if (printGenericCommandUsage) {
1287                ToolRunner.printGenericCommandUsage(out);
1288              }
1289              return true;
1290            }
1291          } catch (ParseException pe) {
1292            return false;
1293          }
1294        }
1295        return false;
1296      }
1297      
1298      /**
1299       * Get DFS_NAMENODE_INVALIDATE_WORK_PCT_PER_ITERATION from configuration.
1300       * 
1301       * @param conf Configuration
1302       * @return Value of DFS_NAMENODE_INVALIDATE_WORK_PCT_PER_ITERATION
1303       */
1304      public static float getInvalidateWorkPctPerIteration(Configuration conf) {
1305        float blocksInvalidateWorkPct = conf.getFloat(
1306            DFSConfigKeys.DFS_NAMENODE_INVALIDATE_WORK_PCT_PER_ITERATION,
1307            DFSConfigKeys.DFS_NAMENODE_INVALIDATE_WORK_PCT_PER_ITERATION_DEFAULT);
1308        Preconditions.checkArgument(
1309            (blocksInvalidateWorkPct > 0 && blocksInvalidateWorkPct <= 1.0f),
1310            DFSConfigKeys.DFS_NAMENODE_INVALIDATE_WORK_PCT_PER_ITERATION +
1311            " = '" + blocksInvalidateWorkPct + "' is invalid. " +
1312            "It should be a positive, non-zero float value, not greater than 1.0f, " +
1313            "to indicate a percentage.");
1314        return blocksInvalidateWorkPct;
1315      }
1316    
1317      /**
1318       * Get DFS_NAMENODE_REPLICATION_WORK_MULTIPLIER_PER_ITERATION from
1319       * configuration.
1320       * 
1321       * @param conf Configuration
1322       * @return Value of DFS_NAMENODE_REPLICATION_WORK_MULTIPLIER_PER_ITERATION
1323       */
1324      public static int getReplWorkMultiplier(Configuration conf) {
1325        int blocksReplWorkMultiplier = conf.getInt(
1326                DFSConfigKeys.DFS_NAMENODE_REPLICATION_WORK_MULTIPLIER_PER_ITERATION,
1327                DFSConfigKeys.DFS_NAMENODE_REPLICATION_WORK_MULTIPLIER_PER_ITERATION_DEFAULT);
1328        Preconditions.checkArgument(
1329            (blocksReplWorkMultiplier > 0),
1330            DFSConfigKeys.DFS_NAMENODE_REPLICATION_WORK_MULTIPLIER_PER_ITERATION +
1331            " = '" + blocksReplWorkMultiplier + "' is invalid. " +
1332            "It should be a positive, non-zero integer value.");
1333        return blocksReplWorkMultiplier;
1334      }
1335      
1336      /**
1337       * Get SPNEGO keytab Key from configuration
1338       * 
1339       * @param conf
1340       *          Configuration
1341       * @param defaultKey
1342       * @return DFS_WEB_AUTHENTICATION_KERBEROS_KEYTAB_KEY if the key is not empty
1343       *         else return defaultKey
1344       */
1345      public static String getSpnegoKeytabKey(Configuration conf, String defaultKey) {
1346        String value = 
1347            conf.get(DFSConfigKeys.DFS_WEB_AUTHENTICATION_KERBEROS_KEYTAB_KEY);
1348        return (value == null || value.isEmpty()) ?
1349            defaultKey : DFSConfigKeys.DFS_WEB_AUTHENTICATION_KERBEROS_KEYTAB_KEY;
1350      }
1351    }