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 */
018package org.apache.hadoop.hdfs;
019
020import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_SIZE_DEFAULT;
021import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_SIZE_KEY;
022import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BYTES_PER_CHECKSUM_DEFAULT;
023import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BYTES_PER_CHECKSUM_KEY;
024import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_DEFAULT;
025import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_KEY;
026import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_RETRIES_DEFAULT;
027import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_RETRIES_KEY;
028import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CACHED_CONN_RETRY_DEFAULT;
029import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CACHED_CONN_RETRY_KEY;
030import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT;
031import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY;
032import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT;
033import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY;
034import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT;
035import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY;
036import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_DEFAULT;
037import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_KEY;
038import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_READ_PREFETCH_SIZE_KEY;
039import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_RETRY_WINDOW_BASE;
040import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_CAPACITY_DEFAULT;
041import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_CAPACITY_KEY;
042import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_DEFAULT;
043import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_KEY;
044import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_TIMEOUT_KEY;
045import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_USE_LEGACY_BLOCKREADER;
046import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_USE_LEGACY_BLOCKREADER_DEFAULT;
047import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_WRITE_PACKET_SIZE_DEFAULT;
048import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_WRITE_PACKET_SIZE_KEY;
049import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_DATANODE_SOCKET_WRITE_TIMEOUT_KEY;
050import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_REPLICATION_DEFAULT;
051import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_REPLICATION_KEY;
052import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_USE_DN_HOSTNAME;
053import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_USE_DN_HOSTNAME_DEFAULT;
054
055import java.io.BufferedOutputStream;
056import java.io.DataInputStream;
057import java.io.DataOutputStream;
058import java.io.FileNotFoundException;
059import java.io.IOException;
060import java.io.InputStream;
061import java.io.OutputStream;
062import java.net.InetAddress;
063import java.net.InetSocketAddress;
064import java.net.NetworkInterface;
065import java.net.Socket;
066import java.net.SocketException;
067import java.net.SocketAddress;
068import java.net.URI;
069import java.net.UnknownHostException;
070import java.util.ArrayList;
071import java.util.Collections;
072import java.util.EnumSet;
073import java.util.HashMap;
074import java.util.LinkedHashMap;
075import java.util.List;
076import java.util.Map;
077import java.util.Random;
078
079import javax.net.SocketFactory;
080
081import org.apache.commons.logging.Log;
082import org.apache.commons.logging.LogFactory;
083import org.apache.hadoop.classification.InterfaceAudience;
084import org.apache.hadoop.conf.Configuration;
085import org.apache.hadoop.fs.BlockLocation;
086import org.apache.hadoop.fs.BlockStorageLocation;
087import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
088import org.apache.hadoop.fs.ContentSummary;
089import org.apache.hadoop.fs.CreateFlag;
090import org.apache.hadoop.fs.FileAlreadyExistsException;
091import org.apache.hadoop.fs.FileSystem;
092import org.apache.hadoop.fs.FsServerDefaults;
093import org.apache.hadoop.fs.FsStatus;
094import org.apache.hadoop.fs.HdfsBlockLocation;
095import org.apache.hadoop.fs.InvalidPathException;
096import org.apache.hadoop.fs.MD5MD5CRC32CastagnoliFileChecksum;
097import org.apache.hadoop.fs.MD5MD5CRC32FileChecksum;
098import org.apache.hadoop.fs.MD5MD5CRC32GzipFileChecksum;
099import org.apache.hadoop.fs.Options;
100import org.apache.hadoop.fs.Options.ChecksumOpt;
101import org.apache.hadoop.fs.ParentNotDirectoryException;
102import org.apache.hadoop.fs.Path;
103import org.apache.hadoop.fs.UnresolvedLinkException;
104import org.apache.hadoop.fs.VolumeId;
105import org.apache.hadoop.fs.permission.FsPermission;
106import org.apache.hadoop.hdfs.client.HdfsDataInputStream;
107import org.apache.hadoop.hdfs.client.HdfsDataOutputStream;
108import org.apache.hadoop.hdfs.protocol.ClientProtocol;
109import org.apache.hadoop.hdfs.protocol.CorruptFileBlocks;
110import org.apache.hadoop.hdfs.protocol.DSQuotaExceededException;
111import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
112import org.apache.hadoop.hdfs.protocol.DirectoryListing;
113import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
114import org.apache.hadoop.hdfs.protocol.HdfsBlocksMetadata;
115import org.apache.hadoop.hdfs.protocol.HdfsConstants;
116import org.apache.hadoop.hdfs.protocol.HdfsConstants.DatanodeReportType;
117import org.apache.hadoop.hdfs.protocol.HdfsConstants.SafeModeAction;
118import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
119import org.apache.hadoop.hdfs.protocol.LocatedBlock;
120import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
121import org.apache.hadoop.hdfs.protocol.NSQuotaExceededException;
122import org.apache.hadoop.hdfs.protocol.UnresolvedPathException;
123import org.apache.hadoop.hdfs.protocol.datatransfer.DataTransferEncryptor;
124import org.apache.hadoop.hdfs.protocol.datatransfer.IOStreamPair;
125import org.apache.hadoop.hdfs.protocol.datatransfer.Op;
126import org.apache.hadoop.hdfs.protocol.datatransfer.ReplaceDatanodeOnFailure;
127import org.apache.hadoop.hdfs.protocol.datatransfer.Sender;
128import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.BlockOpResponseProto;
129import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.OpBlockChecksumResponseProto;
130import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.Status;
131import org.apache.hadoop.hdfs.security.token.block.DataEncryptionKey;
132import org.apache.hadoop.hdfs.protocolPB.PBHelper;
133import org.apache.hadoop.hdfs.security.token.block.BlockTokenIdentifier;
134import org.apache.hadoop.hdfs.security.token.block.InvalidBlockTokenException;
135import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
136import org.apache.hadoop.hdfs.server.common.HdfsServerConstants;
137import org.apache.hadoop.hdfs.server.namenode.NameNode;
138import org.apache.hadoop.hdfs.server.namenode.SafeModeException;
139import org.apache.hadoop.io.DataOutputBuffer;
140import org.apache.hadoop.io.EnumSetWritable;
141import org.apache.hadoop.io.IOUtils;
142import org.apache.hadoop.io.MD5Hash;
143import org.apache.hadoop.io.Text;
144import org.apache.hadoop.ipc.Client;
145import org.apache.hadoop.ipc.RPC;
146import org.apache.hadoop.ipc.RemoteException;
147import org.apache.hadoop.net.DNS;
148import org.apache.hadoop.net.NetUtils;
149import org.apache.hadoop.security.AccessControlException;
150import org.apache.hadoop.security.UserGroupInformation;
151import org.apache.hadoop.security.token.SecretManager.InvalidToken;
152import org.apache.hadoop.security.token.Token;
153import org.apache.hadoop.security.token.TokenRenewer;
154import org.apache.hadoop.util.DataChecksum;
155import org.apache.hadoop.util.DataChecksum.Type;
156import org.apache.hadoop.util.Progressable;
157import org.apache.hadoop.util.Time;
158
159import com.google.common.annotations.VisibleForTesting;
160import com.google.common.base.Joiner;
161import com.google.common.base.Preconditions;
162import com.google.common.net.InetAddresses;
163
164/********************************************************
165 * DFSClient can connect to a Hadoop Filesystem and 
166 * perform basic file tasks.  It uses the ClientProtocol
167 * to communicate with a NameNode daemon, and connects 
168 * directly to DataNodes to read/write block data.
169 *
170 * Hadoop DFS users should obtain an instance of 
171 * DistributedFileSystem, which uses DFSClient to handle
172 * filesystem tasks.
173 *
174 ********************************************************/
175@InterfaceAudience.Private
176public class DFSClient implements java.io.Closeable {
177  public static final Log LOG = LogFactory.getLog(DFSClient.class);
178  public static final long SERVER_DEFAULTS_VALIDITY_PERIOD = 60 * 60 * 1000L; // 1 hour
179  static final int TCP_WINDOW_SIZE = 128 * 1024; // 128 KB
180  final ClientProtocol namenode;
181  /* The service used for delegation tokens */
182  private Text dtService;
183
184  final UserGroupInformation ugi;
185  volatile boolean clientRunning = true;
186  volatile long lastLeaseRenewal;
187  private volatile FsServerDefaults serverDefaults;
188  private volatile long serverDefaultsLastUpdate;
189  final String clientName;
190  Configuration conf;
191  SocketFactory socketFactory;
192  final ReplaceDatanodeOnFailure dtpReplaceDatanodeOnFailure;
193  final FileSystem.Statistics stats;
194  final int hdfsTimeout;    // timeout value for a DFS operation.
195  private final String authority;
196  final SocketCache socketCache;
197  final Conf dfsClientConf;
198  private Random r = new Random();
199  private SocketAddress[] localInterfaceAddrs;
200  private DataEncryptionKey encryptionKey;
201
202  /**
203   * DFSClient configuration 
204   */
205  static class Conf {
206    final int maxFailoverAttempts;
207    final int failoverSleepBaseMillis;
208    final int failoverSleepMaxMillis;
209    final int maxBlockAcquireFailures;
210    final int confTime;
211    final int ioBufferSize;
212    final ChecksumOpt defaultChecksumOpt;
213    final int writePacketSize;
214    final int socketTimeout;
215    final int socketCacheCapacity;
216    final long socketCacheExpiry;
217    /** Wait time window (in msec) if BlockMissingException is caught */
218    final int timeWindow;
219    final int nCachedConnRetry;
220    final int nBlockWriteRetry;
221    final int nBlockWriteLocateFollowingRetry;
222    final long defaultBlockSize;
223    final long prefetchSize;
224    final short defaultReplication;
225    final String taskId;
226    final FsPermission uMask;
227    final boolean useLegacyBlockReader;
228    final boolean connectToDnViaHostname;
229    final boolean getHdfsBlocksMetadataEnabled;
230    final int getFileBlockStorageLocationsNumThreads;
231    final int getFileBlockStorageLocationsTimeout;
232
233    Conf(Configuration conf) {
234      maxFailoverAttempts = conf.getInt(
235          DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY,
236          DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT);
237      failoverSleepBaseMillis = conf.getInt(
238          DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY,
239          DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT);
240      failoverSleepMaxMillis = conf.getInt(
241          DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY,
242          DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT);
243
244      maxBlockAcquireFailures = conf.getInt(
245          DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_KEY,
246          DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_DEFAULT);
247      confTime = conf.getInt(DFS_DATANODE_SOCKET_WRITE_TIMEOUT_KEY,
248          HdfsServerConstants.WRITE_TIMEOUT);
249      ioBufferSize = conf.getInt(
250          CommonConfigurationKeysPublic.IO_FILE_BUFFER_SIZE_KEY,
251          CommonConfigurationKeysPublic.IO_FILE_BUFFER_SIZE_DEFAULT);
252      defaultChecksumOpt = getChecksumOptFromConf(conf);
253      socketTimeout = conf.getInt(DFS_CLIENT_SOCKET_TIMEOUT_KEY,
254          HdfsServerConstants.READ_TIMEOUT);
255      /** dfs.write.packet.size is an internal config variable */
256      writePacketSize = conf.getInt(DFS_CLIENT_WRITE_PACKET_SIZE_KEY,
257          DFS_CLIENT_WRITE_PACKET_SIZE_DEFAULT);
258      defaultBlockSize = conf.getLongBytes(DFS_BLOCK_SIZE_KEY,
259          DFS_BLOCK_SIZE_DEFAULT);
260      defaultReplication = (short) conf.getInt(
261          DFS_REPLICATION_KEY, DFS_REPLICATION_DEFAULT);
262      taskId = conf.get("mapreduce.task.attempt.id", "NONMAPREDUCE");
263      socketCacheCapacity = conf.getInt(DFS_CLIENT_SOCKET_CACHE_CAPACITY_KEY,
264          DFS_CLIENT_SOCKET_CACHE_CAPACITY_DEFAULT);
265      socketCacheExpiry = conf.getLong(DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_KEY,
266          DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_DEFAULT);
267      prefetchSize = conf.getLong(DFS_CLIENT_READ_PREFETCH_SIZE_KEY,
268          10 * defaultBlockSize);
269      timeWindow = conf
270          .getInt(DFS_CLIENT_RETRY_WINDOW_BASE, 3000);
271      nCachedConnRetry = conf.getInt(DFS_CLIENT_CACHED_CONN_RETRY_KEY,
272          DFS_CLIENT_CACHED_CONN_RETRY_DEFAULT);
273      nBlockWriteRetry = conf.getInt(DFS_CLIENT_BLOCK_WRITE_RETRIES_KEY,
274          DFS_CLIENT_BLOCK_WRITE_RETRIES_DEFAULT);
275      nBlockWriteLocateFollowingRetry = conf
276          .getInt(DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_KEY,
277              DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_DEFAULT);
278      uMask = FsPermission.getUMask(conf);
279      useLegacyBlockReader = conf.getBoolean(
280          DFS_CLIENT_USE_LEGACY_BLOCKREADER,
281          DFS_CLIENT_USE_LEGACY_BLOCKREADER_DEFAULT);
282      connectToDnViaHostname = conf.getBoolean(DFS_CLIENT_USE_DN_HOSTNAME,
283          DFS_CLIENT_USE_DN_HOSTNAME_DEFAULT);
284      getHdfsBlocksMetadataEnabled = conf.getBoolean(
285          DFSConfigKeys.DFS_HDFS_BLOCKS_METADATA_ENABLED, 
286          DFSConfigKeys.DFS_HDFS_BLOCKS_METADATA_ENABLED_DEFAULT);
287      getFileBlockStorageLocationsNumThreads = conf.getInt(
288          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_NUM_THREADS,
289          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_NUM_THREADS_DEFAULT);
290      getFileBlockStorageLocationsTimeout = conf.getInt(
291          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_TIMEOUT,
292          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_TIMEOUT_DEFAULT);
293    }
294
295    private DataChecksum.Type getChecksumType(Configuration conf) {
296      final String checksum = conf.get(
297          DFSConfigKeys.DFS_CHECKSUM_TYPE_KEY,
298          DFSConfigKeys.DFS_CHECKSUM_TYPE_DEFAULT);
299      try {
300        return DataChecksum.Type.valueOf(checksum);
301      } catch(IllegalArgumentException iae) {
302        LOG.warn("Bad checksum type: " + checksum + ". Using default "
303            + DFSConfigKeys.DFS_CHECKSUM_TYPE_DEFAULT);
304        return DataChecksum.Type.valueOf(
305            DFSConfigKeys.DFS_CHECKSUM_TYPE_DEFAULT); 
306      }
307    }
308
309    // Construct a checksum option from conf
310    private ChecksumOpt getChecksumOptFromConf(Configuration conf) {
311      DataChecksum.Type type = getChecksumType(conf);
312      int bytesPerChecksum = conf.getInt(DFS_BYTES_PER_CHECKSUM_KEY,
313          DFS_BYTES_PER_CHECKSUM_DEFAULT);
314      return new ChecksumOpt(type, bytesPerChecksum);
315    }
316
317    // create a DataChecksum with the default option.
318    private DataChecksum createChecksum() throws IOException {
319      return createChecksum(null);
320    }
321
322    private DataChecksum createChecksum(ChecksumOpt userOpt) 
323        throws IOException {
324      // Fill in any missing field with the default.
325      ChecksumOpt myOpt = ChecksumOpt.processChecksumOpt(
326          defaultChecksumOpt, userOpt);
327      DataChecksum dataChecksum = DataChecksum.newDataChecksum(
328          myOpt.getChecksumType(),
329          myOpt.getBytesPerChecksum());
330      if (dataChecksum == null) {
331        throw new IOException("Invalid checksum type specified: "
332            + myOpt.getChecksumType().name());
333      }
334      return dataChecksum;
335    }
336  }
337 
338  Conf getConf() {
339    return dfsClientConf;
340  }
341  
342  /**
343   * A map from file names to {@link DFSOutputStream} objects
344   * that are currently being written by this client.
345   * Note that a file can only be written by a single client.
346   */
347  private final Map<String, DFSOutputStream> filesBeingWritten
348      = new HashMap<String, DFSOutputStream>();
349
350  private boolean shortCircuitLocalReads;
351  
352  /**
353   * Same as this(NameNode.getAddress(conf), conf);
354   * @see #DFSClient(InetSocketAddress, Configuration)
355   * @deprecated Deprecated at 0.21
356   */
357  @Deprecated
358  public DFSClient(Configuration conf) throws IOException {
359    this(NameNode.getAddress(conf), conf);
360  }
361  
362  public DFSClient(InetSocketAddress address, Configuration conf) throws IOException {
363    this(NameNode.getUri(address), conf);
364  }
365
366  /**
367   * Same as this(nameNodeUri, conf, null);
368   * @see #DFSClient(URI, Configuration, FileSystem.Statistics)
369   */
370  public DFSClient(URI nameNodeUri, Configuration conf
371      ) throws IOException {
372    this(nameNodeUri, conf, null);
373  }
374
375  /**
376   * Same as this(nameNodeUri, null, conf, stats);
377   * @see #DFSClient(URI, ClientProtocol, Configuration, FileSystem.Statistics) 
378   */
379  public DFSClient(URI nameNodeUri, Configuration conf,
380                   FileSystem.Statistics stats)
381    throws IOException {
382    this(nameNodeUri, null, conf, stats);
383  }
384  
385  /** 
386   * Create a new DFSClient connected to the given nameNodeUri or rpcNamenode.
387   * Exactly one of nameNodeUri or rpcNamenode must be null.
388   */
389  DFSClient(URI nameNodeUri, ClientProtocol rpcNamenode,
390      Configuration conf, FileSystem.Statistics stats)
391    throws IOException {
392    // Copy only the required DFSClient configuration
393    this.dfsClientConf = new Conf(conf);
394    this.conf = conf;
395    this.stats = stats;
396    this.socketFactory = NetUtils.getSocketFactory(conf, ClientProtocol.class);
397    this.dtpReplaceDatanodeOnFailure = ReplaceDatanodeOnFailure.get(conf);
398
399    // The hdfsTimeout is currently the same as the ipc timeout 
400    this.hdfsTimeout = Client.getTimeout(conf);
401    this.ugi = UserGroupInformation.getCurrentUser();
402    
403    this.authority = nameNodeUri == null? "null": nameNodeUri.getAuthority();
404    this.clientName = "DFSClient_" + dfsClientConf.taskId + "_" + 
405        DFSUtil.getRandom().nextInt()  + "_" + Thread.currentThread().getId();
406    
407    if (rpcNamenode != null) {
408      // This case is used for testing.
409      Preconditions.checkArgument(nameNodeUri == null);
410      this.namenode = rpcNamenode;
411      dtService = null;
412    } else {
413      Preconditions.checkArgument(nameNodeUri != null,
414          "null URI");
415      NameNodeProxies.ProxyAndInfo<ClientProtocol> proxyInfo =
416        NameNodeProxies.createProxy(conf, nameNodeUri, ClientProtocol.class);
417      this.dtService = proxyInfo.getDelegationTokenService();
418      this.namenode = proxyInfo.getProxy();
419    }
420
421    // read directly from the block file if configured.
422    this.shortCircuitLocalReads = conf.getBoolean(
423        DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_KEY,
424        DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_DEFAULT);
425    if (LOG.isDebugEnabled()) {
426      LOG.debug("Short circuit read is " + shortCircuitLocalReads);
427    }
428    String localInterfaces[] =
429      conf.getTrimmedStrings(DFSConfigKeys.DFS_CLIENT_LOCAL_INTERFACES);
430    localInterfaceAddrs = getLocalInterfaceAddrs(localInterfaces);
431    if (LOG.isDebugEnabled() && 0 != localInterfaces.length) {
432      LOG.debug("Using local interfaces [" +
433      Joiner.on(',').join(localInterfaces)+ "] with addresses [" +
434      Joiner.on(',').join(localInterfaceAddrs) + "]");
435    }
436    
437    this.socketCache = SocketCache.getInstance(dfsClientConf.socketCacheCapacity, dfsClientConf.socketCacheExpiry);
438  }
439
440  /**
441   * Return the socket addresses to use with each configured
442   * local interface. Local interfaces may be specified by IP
443   * address, IP address range using CIDR notation, interface
444   * name (e.g. eth0) or sub-interface name (e.g. eth0:0).
445   * The socket addresses consist of the IPs for the interfaces
446   * and the ephemeral port (port 0). If an IP, IP range, or
447   * interface name matches an interface with sub-interfaces
448   * only the IP of the interface is used. Sub-interfaces can
449   * be used by specifying them explicitly (by IP or name).
450   * 
451   * @return SocketAddresses for the configured local interfaces,
452   *    or an empty array if none are configured
453   * @throws UnknownHostException if a given interface name is invalid
454   */
455  private static SocketAddress[] getLocalInterfaceAddrs(
456      String interfaceNames[]) throws UnknownHostException {
457    List<SocketAddress> localAddrs = new ArrayList<SocketAddress>();
458    for (String interfaceName : interfaceNames) {
459      if (InetAddresses.isInetAddress(interfaceName)) {
460        localAddrs.add(new InetSocketAddress(interfaceName, 0));
461      } else if (NetUtils.isValidSubnet(interfaceName)) {
462        for (InetAddress addr : NetUtils.getIPs(interfaceName, false)) {
463          localAddrs.add(new InetSocketAddress(addr, 0));
464        }
465      } else {
466        for (String ip : DNS.getIPs(interfaceName, false)) {
467          localAddrs.add(new InetSocketAddress(ip, 0));
468        }
469      }
470    }
471    return localAddrs.toArray(new SocketAddress[localAddrs.size()]);
472  }
473
474  /**
475   * Select one of the configured local interfaces at random. We use a random
476   * interface because other policies like round-robin are less effective
477   * given that we cache connections to datanodes.
478   *
479   * @return one of the local interface addresses at random, or null if no
480   *    local interfaces are configured
481   */
482  SocketAddress getRandomLocalInterfaceAddr() {
483    if (localInterfaceAddrs.length == 0) {
484      return null;
485    }
486    final int idx = r.nextInt(localInterfaceAddrs.length);
487    final SocketAddress addr = localInterfaceAddrs[idx];
488    if (LOG.isDebugEnabled()) {
489      LOG.debug("Using local interface " + addr);
490    }
491    return addr;
492  }
493
494  /**
495   * Return the number of times the client should go back to the namenode
496   * to retrieve block locations when reading.
497   */
498  int getMaxBlockAcquireFailures() {
499    return dfsClientConf.maxBlockAcquireFailures;
500  }
501
502  /**
503   * Return the timeout that clients should use when writing to datanodes.
504   * @param numNodes the number of nodes in the pipeline.
505   */
506  int getDatanodeWriteTimeout(int numNodes) {
507    return (dfsClientConf.confTime > 0) ?
508      (dfsClientConf.confTime + HdfsServerConstants.WRITE_TIMEOUT_EXTENSION * numNodes) : 0;
509  }
510
511  int getDatanodeReadTimeout(int numNodes) {
512    return dfsClientConf.socketTimeout > 0 ?
513        (HdfsServerConstants.READ_TIMEOUT_EXTENSION * numNodes +
514            dfsClientConf.socketTimeout) : 0;
515  }
516  
517  int getHdfsTimeout() {
518    return hdfsTimeout;
519  }
520  
521  String getClientName() {
522    return clientName;
523  }
524
525  /**
526   * @return whether the client should use hostnames instead of IPs
527   *    when connecting to DataNodes
528   */
529  boolean connectToDnViaHostname() {
530    return dfsClientConf.connectToDnViaHostname;
531  }
532
533  void checkOpen() throws IOException {
534    if (!clientRunning) {
535      IOException result = new IOException("Filesystem closed");
536      throw result;
537    }
538  }
539
540  /** Return the lease renewer instance. The renewer thread won't start
541   *  until the first output stream is created. The same instance will
542   *  be returned until all output streams are closed.
543   */
544  public LeaseRenewer getLeaseRenewer() throws IOException {
545      return LeaseRenewer.getInstance(authority, ugi, this);
546  }
547
548  /** Get a lease and start automatic renewal */
549  private void beginFileLease(final String src, final DFSOutputStream out) 
550      throws IOException {
551    getLeaseRenewer().put(src, out, this);
552  }
553
554  /** Stop renewal of lease for the file. */
555  void endFileLease(final String src) throws IOException {
556    getLeaseRenewer().closeFile(src, this);
557  }
558    
559
560  /** Put a file. Only called from LeaseRenewer, where proper locking is
561   *  enforced to consistently update its local dfsclients array and 
562   *  client's filesBeingWritten map.
563   */
564  void putFileBeingWritten(final String src, final DFSOutputStream out) {
565    synchronized(filesBeingWritten) {
566      filesBeingWritten.put(src, out);
567      // update the last lease renewal time only when there was no
568      // writes. once there is one write stream open, the lease renewer
569      // thread keeps it updated well with in anyone's expiration time.
570      if (lastLeaseRenewal == 0) {
571        updateLastLeaseRenewal();
572      }
573    }
574  }
575
576  /** Remove a file. Only called from LeaseRenewer. */
577  void removeFileBeingWritten(final String src) {
578    synchronized(filesBeingWritten) {
579      filesBeingWritten.remove(src);
580      if (filesBeingWritten.isEmpty()) {
581        lastLeaseRenewal = 0;
582      }
583    }
584  }
585
586  /** Is file-being-written map empty? */
587  boolean isFilesBeingWrittenEmpty() {
588    synchronized(filesBeingWritten) {
589      return filesBeingWritten.isEmpty();
590    }
591  }
592  
593  /** @return true if the client is running */
594  boolean isClientRunning() {
595    return clientRunning;
596  }
597
598  long getLastLeaseRenewal() {
599    return lastLeaseRenewal;
600  }
601
602  void updateLastLeaseRenewal() {
603    synchronized(filesBeingWritten) {
604      if (filesBeingWritten.isEmpty()) {
605        return;
606      }
607      lastLeaseRenewal = Time.now();
608    }
609  }
610
611  /**
612   * Renew leases.
613   * @return true if lease was renewed. May return false if this
614   * client has been closed or has no files open.
615   **/
616  boolean renewLease() throws IOException {
617    if (clientRunning && !isFilesBeingWrittenEmpty()) {
618      try {
619        namenode.renewLease(clientName);
620        updateLastLeaseRenewal();
621        return true;
622      } catch (IOException e) {
623        // Abort if the lease has already expired. 
624        final long elapsed = Time.now() - getLastLeaseRenewal();
625        if (elapsed > HdfsConstants.LEASE_HARDLIMIT_PERIOD) {
626          LOG.warn("Failed to renew lease for " + clientName + " for "
627              + (elapsed/1000) + " seconds (>= soft-limit ="
628              + (HdfsConstants.LEASE_HARDLIMIT_PERIOD/1000) + " seconds.) "
629              + "Closing all files being written ...", e);
630          closeAllFilesBeingWritten(true);
631        } else {
632          // Let the lease renewer handle it and retry.
633          throw e;
634        }
635      }
636    }
637    return false;
638  }
639  
640  /**
641   * Close connections the Namenode.
642   */
643  void closeConnectionToNamenode() {
644    RPC.stopProxy(namenode);
645  }
646  
647  /** Abort and release resources held.  Ignore all errors. */
648  void abort() {
649    clientRunning = false;
650    closeAllFilesBeingWritten(true);
651
652    try {
653      // remove reference to this client and stop the renewer,
654      // if there is no more clients under the renewer.
655      getLeaseRenewer().closeClient(this);
656    } catch (IOException ioe) {
657       LOG.info("Exception occurred while aborting the client " + ioe);
658    }
659    closeConnectionToNamenode();
660  }
661
662  /** Close/abort all files being written. */
663  private void closeAllFilesBeingWritten(final boolean abort) {
664    for(;;) {
665      final String src;
666      final DFSOutputStream out;
667      synchronized(filesBeingWritten) {
668        if (filesBeingWritten.isEmpty()) {
669          return;
670        }
671        src = filesBeingWritten.keySet().iterator().next();
672        out = filesBeingWritten.remove(src);
673      }
674      if (out != null) {
675        try {
676          if (abort) {
677            out.abort();
678          } else {
679            out.close();
680          }
681        } catch(IOException ie) {
682          LOG.error("Failed to " + (abort? "abort": "close") + " file " + src,
683              ie);
684        }
685      }
686    }
687  }
688
689  /**
690   * Close the file system, abandoning all of the leases and files being
691   * created and close connections to the namenode.
692   */
693  @Override
694  public synchronized void close() throws IOException {
695    if(clientRunning) {
696      closeAllFilesBeingWritten(false);
697      clientRunning = false;
698      getLeaseRenewer().closeClient(this);
699      // close connections to the namenode
700      closeConnectionToNamenode();
701    }
702  }
703
704  /**
705   * Get the default block size for this cluster
706   * @return the default block size in bytes
707   */
708  public long getDefaultBlockSize() {
709    return dfsClientConf.defaultBlockSize;
710  }
711    
712  /**
713   * @see ClientProtocol#getPreferredBlockSize(String)
714   */
715  public long getBlockSize(String f) throws IOException {
716    try {
717      return namenode.getPreferredBlockSize(f);
718    } catch (IOException ie) {
719      LOG.warn("Problem getting block size", ie);
720      throw ie;
721    }
722  }
723
724  /**
725   * Get server default values for a number of configuration params.
726   * @see ClientProtocol#getServerDefaults()
727   */
728  public FsServerDefaults getServerDefaults() throws IOException {
729    long now = Time.now();
730    if (now - serverDefaultsLastUpdate > SERVER_DEFAULTS_VALIDITY_PERIOD) {
731      serverDefaults = namenode.getServerDefaults();
732      serverDefaultsLastUpdate = now;
733    }
734    return serverDefaults;
735  }
736  
737  /**
738   * Get a canonical token service name for this client's tokens.  Null should
739   * be returned if the client is not using tokens.
740   * @return the token service for the client
741   */
742  @InterfaceAudience.LimitedPrivate( { "HDFS" }) 
743  public String getCanonicalServiceName() {
744    return (dtService != null) ? dtService.toString() : null;
745  }
746  
747  /**
748   * @see ClientProtocol#getDelegationToken(Text)
749   */
750  public Token<DelegationTokenIdentifier> getDelegationToken(Text renewer)
751      throws IOException {
752    assert dtService != null;
753    Token<DelegationTokenIdentifier> token =
754      namenode.getDelegationToken(renewer);
755    token.setService(this.dtService);
756
757    LOG.info("Created " + DelegationTokenIdentifier.stringifyToken(token));
758    return token;
759  }
760
761  /**
762   * Renew a delegation token
763   * @param token the token to renew
764   * @return the new expiration time
765   * @throws InvalidToken
766   * @throws IOException
767   * @deprecated Use Token.renew instead.
768   */
769  public long renewDelegationToken(Token<DelegationTokenIdentifier> token)
770      throws InvalidToken, IOException {
771    LOG.info("Renewing " + DelegationTokenIdentifier.stringifyToken(token));
772    try {
773      return token.renew(conf);
774    } catch (InterruptedException ie) {                                       
775      throw new RuntimeException("caught interrupted", ie);
776    } catch (RemoteException re) {
777      throw re.unwrapRemoteException(InvalidToken.class,
778                                     AccessControlException.class);
779    }
780  }
781
782  /**
783   * Get {@link BlockReader} for short circuited local reads.
784   */
785  static BlockReader getLocalBlockReader(Configuration conf,
786      String src, ExtendedBlock blk, Token<BlockTokenIdentifier> accessToken,
787      DatanodeInfo chosenNode, int socketTimeout, long offsetIntoBlock,
788      boolean connectToDnViaHostname) throws InvalidToken, IOException {
789    try {
790      return BlockReaderLocal.newBlockReader(conf, src, blk, accessToken,
791          chosenNode, socketTimeout, offsetIntoBlock, blk.getNumBytes()
792              - offsetIntoBlock, connectToDnViaHostname);
793    } catch (RemoteException re) {
794      throw re.unwrapRemoteException(InvalidToken.class,
795          AccessControlException.class);
796    }
797  }
798  
799  private static Map<String, Boolean> localAddrMap = Collections
800      .synchronizedMap(new HashMap<String, Boolean>());
801  
802  private static boolean isLocalAddress(InetSocketAddress targetAddr) {
803    InetAddress addr = targetAddr.getAddress();
804    Boolean cached = localAddrMap.get(addr.getHostAddress());
805    if (cached != null) {
806      if (LOG.isTraceEnabled()) {
807        LOG.trace("Address " + targetAddr +
808                  (cached ? " is local" : " is not local"));
809      }
810      return cached;
811    }
812
813    // Check if the address is any local or loop back
814    boolean local = addr.isAnyLocalAddress() || addr.isLoopbackAddress();
815
816    // Check if the address is defined on any interface
817    if (!local) {
818      try {
819        local = NetworkInterface.getByInetAddress(addr) != null;
820      } catch (SocketException e) {
821        local = false;
822      }
823    }
824    if (LOG.isTraceEnabled()) {
825      LOG.trace("Address " + targetAddr +
826                (local ? " is local" : " is not local"));
827    }
828    localAddrMap.put(addr.getHostAddress(), local);
829    return local;
830  }
831  
832  /**
833   * Should the block access token be refetched on an exception
834   * 
835   * @param ex Exception received
836   * @param targetAddr Target datanode address from where exception was received
837   * @return true if block access token has expired or invalid and it should be
838   *         refetched
839   */
840  private static boolean tokenRefetchNeeded(IOException ex,
841      InetSocketAddress targetAddr) {
842    /*
843     * Get a new access token and retry. Retry is needed in 2 cases. 1) When
844     * both NN and DN re-started while DFSClient holding a cached access token.
845     * 2) In the case that NN fails to update its access key at pre-set interval
846     * (by a wide margin) and subsequently restarts. In this case, DN
847     * re-registers itself with NN and receives a new access key, but DN will
848     * delete the old access key from its memory since it's considered expired
849     * based on the estimated expiration date.
850     */
851    if (ex instanceof InvalidBlockTokenException || ex instanceof InvalidToken) {
852      LOG.info("Access token was invalid when connecting to " + targetAddr
853          + " : " + ex);
854      return true;
855    }
856    return false;
857  }
858  
859  /**
860   * Cancel a delegation token
861   * @param token the token to cancel
862   * @throws InvalidToken
863   * @throws IOException
864   * @deprecated Use Token.cancel instead.
865   */
866  public void cancelDelegationToken(Token<DelegationTokenIdentifier> token)
867      throws InvalidToken, IOException {
868    LOG.info("Cancelling " + DelegationTokenIdentifier.stringifyToken(token));
869    try {
870      token.cancel(conf);
871     } catch (InterruptedException ie) {                                       
872      throw new RuntimeException("caught interrupted", ie);
873    } catch (RemoteException re) {
874      throw re.unwrapRemoteException(InvalidToken.class,
875                                     AccessControlException.class);
876    }
877  }
878  
879  @InterfaceAudience.Private
880  public static class Renewer extends TokenRenewer {
881    
882    static {
883      //Ensure that HDFS Configuration files are loaded before trying to use
884      // the renewer.
885      HdfsConfiguration.init();
886    }
887    
888    @Override
889    public boolean handleKind(Text kind) {
890      return DelegationTokenIdentifier.HDFS_DELEGATION_KIND.equals(kind);
891    }
892
893    @SuppressWarnings("unchecked")
894    @Override
895    public long renew(Token<?> token, Configuration conf) throws IOException {
896      Token<DelegationTokenIdentifier> delToken = 
897        (Token<DelegationTokenIdentifier>) token;
898      ClientProtocol nn = getNNProxy(delToken, conf);
899      try {
900        return nn.renewDelegationToken(delToken);
901      } catch (RemoteException re) {
902        throw re.unwrapRemoteException(InvalidToken.class, 
903                                       AccessControlException.class);
904      }
905    }
906
907    @SuppressWarnings("unchecked")
908    @Override
909    public void cancel(Token<?> token, Configuration conf) throws IOException {
910      Token<DelegationTokenIdentifier> delToken = 
911          (Token<DelegationTokenIdentifier>) token;
912      LOG.info("Cancelling " + 
913               DelegationTokenIdentifier.stringifyToken(delToken));
914      ClientProtocol nn = getNNProxy(delToken, conf);
915      try {
916        nn.cancelDelegationToken(delToken);
917      } catch (RemoteException re) {
918        throw re.unwrapRemoteException(InvalidToken.class,
919            AccessControlException.class);
920      }
921    }
922    
923    private static ClientProtocol getNNProxy(
924        Token<DelegationTokenIdentifier> token, Configuration conf)
925        throws IOException {
926      URI uri = HAUtil.getServiceUriFromToken(token);
927      if (HAUtil.isTokenForLogicalUri(token) &&
928          !HAUtil.isLogicalUri(conf, uri)) {
929        // If the token is for a logical nameservice, but the configuration
930        // we have disagrees about that, we can't actually renew it.
931        // This can be the case in MR, for example, if the RM doesn't
932        // have all of the HA clusters configured in its configuration.
933        throw new IOException("Unable to map logical nameservice URI '" +
934            uri + "' to a NameNode. Local configuration does not have " +
935            "a failover proxy provider configured.");
936      }
937      
938      NameNodeProxies.ProxyAndInfo<ClientProtocol> info =
939        NameNodeProxies.createProxy(conf, uri, ClientProtocol.class);
940      assert info.getDelegationTokenService().equals(token.getService()) :
941        "Returned service '" + info.getDelegationTokenService().toString() +
942        "' doesn't match expected service '" +
943        token.getService().toString() + "'";
944        
945      return info.getProxy();
946    }
947
948    @Override
949    public boolean isManaged(Token<?> token) throws IOException {
950      return true;
951    }
952    
953  }
954
955  /**
956   * Report corrupt blocks that were discovered by the client.
957   * @see ClientProtocol#reportBadBlocks(LocatedBlock[])
958   */
959  public void reportBadBlocks(LocatedBlock[] blocks) throws IOException {
960    namenode.reportBadBlocks(blocks);
961  }
962  
963  public short getDefaultReplication() {
964    return dfsClientConf.defaultReplication;
965  }
966  
967  /*
968   * This is just a wrapper around callGetBlockLocations, but non-static so that
969   * we can stub it out for tests.
970   */
971  @VisibleForTesting
972  public LocatedBlocks getLocatedBlocks(String src, long start, long length)
973      throws IOException {
974    return callGetBlockLocations(namenode, src, start, length);
975  }
976
977  /**
978   * @see ClientProtocol#getBlockLocations(String, long, long)
979   */
980  static LocatedBlocks callGetBlockLocations(ClientProtocol namenode,
981      String src, long start, long length) 
982      throws IOException {
983    try {
984      return namenode.getBlockLocations(src, start, length);
985    } catch(RemoteException re) {
986      throw re.unwrapRemoteException(AccessControlException.class,
987                                     FileNotFoundException.class,
988                                     UnresolvedPathException.class);
989    }
990  }
991
992  /**
993   * Recover a file's lease
994   * @param src a file's path
995   * @return true if the file is already closed
996   * @throws IOException
997   */
998  boolean recoverLease(String src) throws IOException {
999    checkOpen();
1000
1001    try {
1002      return namenode.recoverLease(src, clientName);
1003    } catch (RemoteException re) {
1004      throw re.unwrapRemoteException(FileNotFoundException.class,
1005                                     AccessControlException.class);
1006    }
1007  }
1008
1009  /**
1010   * Get block location info about file
1011   * 
1012   * getBlockLocations() returns a list of hostnames that store 
1013   * data for a specific file region.  It returns a set of hostnames
1014   * for every block within the indicated region.
1015   *
1016   * This function is very useful when writing code that considers
1017   * data-placement when performing operations.  For example, the
1018   * MapReduce system tries to schedule tasks on the same machines
1019   * as the data-block the task processes. 
1020   */
1021  public BlockLocation[] getBlockLocations(String src, long start, 
1022    long length) throws IOException, UnresolvedLinkException {
1023    LocatedBlocks blocks = getLocatedBlocks(src, start, length);
1024    BlockLocation[] locations =  DFSUtil.locatedBlocks2Locations(blocks);
1025    HdfsBlockLocation[] hdfsLocations = new HdfsBlockLocation[locations.length];
1026    for (int i = 0; i < locations.length; i++) {
1027      hdfsLocations[i] = new HdfsBlockLocation(locations[i], blocks.get(i));
1028    }
1029    return hdfsLocations;
1030  }
1031  
1032  /**
1033   * Get block location information about a list of {@link HdfsBlockLocation}.
1034   * Used by {@link DistributedFileSystem#getFileBlockStorageLocations(List)} to
1035   * get {@link BlockStorageLocation}s for blocks returned by
1036   * {@link DistributedFileSystem#getFileBlockLocations(org.apache.hadoop.fs.FileStatus, long, long)}
1037   * .
1038   * 
1039   * This is done by making a round of RPCs to the associated datanodes, asking
1040   * the volume of each block replica. The returned array of
1041   * {@link BlockStorageLocation} expose this information as a
1042   * {@link VolumeId}.
1043   * 
1044   * @param blockLocations
1045   *          target blocks on which to query volume location information
1046   * @return volumeBlockLocations original block array augmented with additional
1047   *         volume location information for each replica.
1048   */
1049  public BlockStorageLocation[] getBlockStorageLocations(
1050      List<BlockLocation> blockLocations) throws IOException,
1051      UnsupportedOperationException, InvalidBlockTokenException {
1052    if (!getConf().getHdfsBlocksMetadataEnabled) {
1053      throw new UnsupportedOperationException("Datanode-side support for " +
1054          "getVolumeBlockLocations() must also be enabled in the client " +
1055          "configuration.");
1056    }
1057    // Downcast blockLocations and fetch out required LocatedBlock(s)
1058    List<LocatedBlock> blocks = new ArrayList<LocatedBlock>();
1059    for (BlockLocation loc : blockLocations) {
1060      if (!(loc instanceof HdfsBlockLocation)) {
1061        throw new ClassCastException("DFSClient#getVolumeBlockLocations " +
1062            "expected to be passed HdfsBlockLocations");
1063      }
1064      HdfsBlockLocation hdfsLoc = (HdfsBlockLocation) loc;
1065      blocks.add(hdfsLoc.getLocatedBlock());
1066    }
1067    
1068    // Re-group the LocatedBlocks to be grouped by datanodes, with the values
1069    // a list of the LocatedBlocks on the datanode.
1070    Map<DatanodeInfo, List<LocatedBlock>> datanodeBlocks = 
1071        new LinkedHashMap<DatanodeInfo, List<LocatedBlock>>();
1072    for (LocatedBlock b : blocks) {
1073      for (DatanodeInfo info : b.getLocations()) {
1074        if (!datanodeBlocks.containsKey(info)) {
1075          datanodeBlocks.put(info, new ArrayList<LocatedBlock>());
1076        }
1077        List<LocatedBlock> l = datanodeBlocks.get(info);
1078        l.add(b);
1079      }
1080    }
1081        
1082    // Make RPCs to the datanodes to get volume locations for its replicas
1083    List<HdfsBlocksMetadata> metadatas = BlockStorageLocationUtil
1084        .queryDatanodesForHdfsBlocksMetadata(conf, datanodeBlocks,
1085            getConf().getFileBlockStorageLocationsNumThreads,
1086            getConf().getFileBlockStorageLocationsTimeout,
1087            getConf().connectToDnViaHostname);
1088    
1089    // Regroup the returned VolumeId metadata to again be grouped by
1090    // LocatedBlock rather than by datanode
1091    Map<LocatedBlock, List<VolumeId>> blockVolumeIds = BlockStorageLocationUtil
1092        .associateVolumeIdsWithBlocks(blocks, datanodeBlocks, metadatas);
1093    
1094    // Combine original BlockLocations with new VolumeId information
1095    BlockStorageLocation[] volumeBlockLocations = BlockStorageLocationUtil
1096        .convertToVolumeBlockLocations(blocks, blockVolumeIds);
1097
1098    return volumeBlockLocations;
1099  }
1100  
1101  public DFSInputStream open(String src) 
1102      throws IOException, UnresolvedLinkException {
1103    return open(src, dfsClientConf.ioBufferSize, true, null);
1104  }
1105
1106  /**
1107   * Create an input stream that obtains a nodelist from the
1108   * namenode, and then reads from all the right places.  Creates
1109   * inner subclass of InputStream that does the right out-of-band
1110   * work.
1111   * @deprecated Use {@link #open(String, int, boolean)} instead.
1112   */
1113  @Deprecated
1114  public DFSInputStream open(String src, int buffersize, boolean verifyChecksum,
1115                             FileSystem.Statistics stats)
1116      throws IOException, UnresolvedLinkException {
1117    return open(src, buffersize, verifyChecksum);
1118  }
1119  
1120
1121  /**
1122   * Create an input stream that obtains a nodelist from the
1123   * namenode, and then reads from all the right places.  Creates
1124   * inner subclass of InputStream that does the right out-of-band
1125   * work.
1126   */
1127  public DFSInputStream open(String src, int buffersize, boolean verifyChecksum)
1128      throws IOException, UnresolvedLinkException {
1129    checkOpen();
1130    //    Get block info from namenode
1131    return new DFSInputStream(this, src, buffersize, verifyChecksum);
1132  }
1133
1134  /**
1135   * Get the namenode associated with this DFSClient object
1136   * @return the namenode associated with this DFSClient object
1137   */
1138  public ClientProtocol getNamenode() {
1139    return namenode;
1140  }
1141  
1142  /**
1143   * Call {@link #create(String, boolean, short, long, Progressable)} with
1144   * default <code>replication</code> and <code>blockSize<code> and null <code>
1145   * progress</code>.
1146   */
1147  public OutputStream create(String src, boolean overwrite) 
1148      throws IOException {
1149    return create(src, overwrite, dfsClientConf.defaultReplication,
1150        dfsClientConf.defaultBlockSize, null);
1151  }
1152    
1153  /**
1154   * Call {@link #create(String, boolean, short, long, Progressable)} with
1155   * default <code>replication</code> and <code>blockSize<code>.
1156   */
1157  public OutputStream create(String src, 
1158                             boolean overwrite,
1159                             Progressable progress) throws IOException {
1160    return create(src, overwrite, dfsClientConf.defaultReplication,
1161        dfsClientConf.defaultBlockSize, progress);
1162  }
1163    
1164  /**
1165   * Call {@link #create(String, boolean, short, long, Progressable)} with
1166   * null <code>progress</code>.
1167   */
1168  public OutputStream create(String src, 
1169                             boolean overwrite, 
1170                             short replication,
1171                             long blockSize) throws IOException {
1172    return create(src, overwrite, replication, blockSize, null);
1173  }
1174
1175  /**
1176   * Call {@link #create(String, boolean, short, long, Progressable, int)}
1177   * with default bufferSize.
1178   */
1179  public OutputStream create(String src, boolean overwrite, short replication,
1180      long blockSize, Progressable progress) throws IOException {
1181    return create(src, overwrite, replication, blockSize, progress,
1182        dfsClientConf.ioBufferSize);
1183  }
1184
1185  /**
1186   * Call {@link #create(String, FsPermission, EnumSet, short, long, 
1187   * Progressable, int, ChecksumOpt)} with default <code>permission</code>
1188   * {@link FsPermission#getFileDefault()}.
1189   * 
1190   * @param src File name
1191   * @param overwrite overwrite an existing file if true
1192   * @param replication replication factor for the file
1193   * @param blockSize maximum block size
1194   * @param progress interface for reporting client progress
1195   * @param buffersize underlying buffersize
1196   * 
1197   * @return output stream
1198   */
1199  public OutputStream create(String src,
1200                             boolean overwrite,
1201                             short replication,
1202                             long blockSize,
1203                             Progressable progress,
1204                             int buffersize)
1205      throws IOException {
1206    return create(src, FsPermission.getFileDefault(),
1207        overwrite ? EnumSet.of(CreateFlag.CREATE, CreateFlag.OVERWRITE)
1208            : EnumSet.of(CreateFlag.CREATE), replication, blockSize, progress,
1209        buffersize, null);
1210  }
1211
1212  /**
1213   * Call {@link #create(String, FsPermission, EnumSet, boolean, short, 
1214   * long, Progressable, int, ChecksumOpt)} with <code>createParent</code>
1215   *  set to true.
1216   */
1217  public DFSOutputStream create(String src, 
1218                             FsPermission permission,
1219                             EnumSet<CreateFlag> flag, 
1220                             short replication,
1221                             long blockSize,
1222                             Progressable progress,
1223                             int buffersize,
1224                             ChecksumOpt checksumOpt)
1225      throws IOException {
1226    return create(src, permission, flag, true,
1227        replication, blockSize, progress, buffersize, checksumOpt);
1228  }
1229
1230  /**
1231   * Create a new dfs file with the specified block replication 
1232   * with write-progress reporting and return an output stream for writing
1233   * into the file.  
1234   * 
1235   * @param src File name
1236   * @param permission The permission of the directory being created.
1237   *          If null, use default permission {@link FsPermission#getFileDefault()}
1238   * @param flag indicates create a new file or create/overwrite an
1239   *          existing file or append to an existing file
1240   * @param createParent create missing parent directory if true
1241   * @param replication block replication
1242   * @param blockSize maximum block size
1243   * @param progress interface for reporting client progress
1244   * @param buffersize underlying buffer size 
1245   * @param checksumOpt checksum options
1246   * 
1247   * @return output stream
1248   * 
1249   * @see ClientProtocol#create(String, FsPermission, String, EnumSetWritable,
1250   * boolean, short, long) for detailed description of exceptions thrown
1251   */
1252  public DFSOutputStream create(String src, 
1253                             FsPermission permission,
1254                             EnumSet<CreateFlag> flag, 
1255                             boolean createParent,
1256                             short replication,
1257                             long blockSize,
1258                             Progressable progress,
1259                             int buffersize,
1260                             ChecksumOpt checksumOpt) throws IOException {
1261    checkOpen();
1262    if (permission == null) {
1263      permission = FsPermission.getFileDefault();
1264    }
1265    FsPermission masked = permission.applyUMask(dfsClientConf.uMask);
1266    if(LOG.isDebugEnabled()) {
1267      LOG.debug(src + ": masked=" + masked);
1268    }
1269    final DFSOutputStream result = DFSOutputStream.newStreamForCreate(this,
1270        src, masked, flag, createParent, replication, blockSize, progress,
1271        buffersize, dfsClientConf.createChecksum(checksumOpt));
1272    beginFileLease(src, result);
1273    return result;
1274  }
1275  
1276  /**
1277   * Append to an existing file if {@link CreateFlag#APPEND} is present
1278   */
1279  private DFSOutputStream primitiveAppend(String src, EnumSet<CreateFlag> flag,
1280      int buffersize, Progressable progress) throws IOException {
1281    if (flag.contains(CreateFlag.APPEND)) {
1282      HdfsFileStatus stat = getFileInfo(src);
1283      if (stat == null) { // No file to append to
1284        // New file needs to be created if create option is present
1285        if (!flag.contains(CreateFlag.CREATE)) {
1286          throw new FileNotFoundException("failed to append to non-existent file "
1287              + src + " on client " + clientName);
1288        }
1289        return null;
1290      }
1291      return callAppend(stat, src, buffersize, progress);
1292    }
1293    return null;
1294  }
1295  
1296  /**
1297   * Same as {{@link #create(String, FsPermission, EnumSet, short, long,
1298   *  Progressable, int, ChecksumOpt)} except that the permission
1299   *  is absolute (ie has already been masked with umask.
1300   */
1301  public DFSOutputStream primitiveCreate(String src, 
1302                             FsPermission absPermission,
1303                             EnumSet<CreateFlag> flag,
1304                             boolean createParent,
1305                             short replication,
1306                             long blockSize,
1307                             Progressable progress,
1308                             int buffersize,
1309                             ChecksumOpt checksumOpt)
1310      throws IOException, UnresolvedLinkException {
1311    checkOpen();
1312    CreateFlag.validate(flag);
1313    DFSOutputStream result = primitiveAppend(src, flag, buffersize, progress);
1314    if (result == null) {
1315      DataChecksum checksum = dfsClientConf.createChecksum(checksumOpt);
1316      result = DFSOutputStream.newStreamForCreate(this, src, absPermission,
1317          flag, createParent, replication, blockSize, progress, buffersize,
1318          checksum);
1319    }
1320    beginFileLease(src, result);
1321    return result;
1322  }
1323  
1324  /**
1325   * Creates a symbolic link.
1326   * 
1327   * @see ClientProtocol#createSymlink(String, String,FsPermission, boolean) 
1328   */
1329  public void createSymlink(String target, String link, boolean createParent)
1330      throws IOException {
1331    try {
1332      FsPermission dirPerm = 
1333          FsPermission.getDefault().applyUMask(dfsClientConf.uMask); 
1334      namenode.createSymlink(target, link, dirPerm, createParent);
1335    } catch (RemoteException re) {
1336      throw re.unwrapRemoteException(AccessControlException.class,
1337                                     FileAlreadyExistsException.class, 
1338                                     FileNotFoundException.class,
1339                                     ParentNotDirectoryException.class,
1340                                     NSQuotaExceededException.class, 
1341                                     DSQuotaExceededException.class,
1342                                     UnresolvedPathException.class);
1343    }
1344  }
1345
1346  /**
1347   * Resolve the *first* symlink, if any, in the path.
1348   * 
1349   * @see ClientProtocol#getLinkTarget(String)
1350   */
1351  public String getLinkTarget(String path) throws IOException { 
1352    checkOpen();
1353    try {
1354      return namenode.getLinkTarget(path);
1355    } catch (RemoteException re) {
1356      throw re.unwrapRemoteException(AccessControlException.class,
1357                                     FileNotFoundException.class);
1358    }
1359  }
1360
1361  /** Method to get stream returned by append call */
1362  private DFSOutputStream callAppend(HdfsFileStatus stat, String src,
1363      int buffersize, Progressable progress) throws IOException {
1364    LocatedBlock lastBlock = null;
1365    try {
1366      lastBlock = namenode.append(src, clientName);
1367    } catch(RemoteException re) {
1368      throw re.unwrapRemoteException(AccessControlException.class,
1369                                     FileNotFoundException.class,
1370                                     SafeModeException.class,
1371                                     DSQuotaExceededException.class,
1372                                     UnsupportedOperationException.class,
1373                                     UnresolvedPathException.class);
1374    }
1375    return DFSOutputStream.newStreamForAppend(this, src, buffersize, progress,
1376        lastBlock, stat, dfsClientConf.createChecksum());
1377  }
1378  
1379  /**
1380   * Append to an existing HDFS file.  
1381   * 
1382   * @param src file name
1383   * @param buffersize buffer size
1384   * @param progress for reporting write-progress; null is acceptable.
1385   * @param statistics file system statistics; null is acceptable.
1386   * @return an output stream for writing into the file
1387   * 
1388   * @see ClientProtocol#append(String, String) 
1389   */
1390  public HdfsDataOutputStream append(final String src, final int buffersize,
1391      final Progressable progress, final FileSystem.Statistics statistics
1392      ) throws IOException {
1393    final DFSOutputStream out = append(src, buffersize, progress);
1394    return new HdfsDataOutputStream(out, statistics, out.getInitialLen());
1395  }
1396
1397  private DFSOutputStream append(String src, int buffersize, Progressable progress) 
1398      throws IOException {
1399    checkOpen();
1400    HdfsFileStatus stat = getFileInfo(src);
1401    if (stat == null) { // No file found
1402      throw new FileNotFoundException("failed to append to non-existent file "
1403          + src + " on client " + clientName);
1404    }
1405    final DFSOutputStream result = callAppend(stat, src, buffersize, progress);
1406    beginFileLease(src, result);
1407    return result;
1408  }
1409
1410  /**
1411   * Set replication for an existing file.
1412   * @param src file name
1413   * @param replication
1414   * 
1415   * @see ClientProtocol#setReplication(String, short)
1416   */
1417  public boolean setReplication(String src, short replication)
1418      throws IOException {
1419    try {
1420      return namenode.setReplication(src, replication);
1421    } catch(RemoteException re) {
1422      throw re.unwrapRemoteException(AccessControlException.class,
1423                                     FileNotFoundException.class,
1424                                     SafeModeException.class,
1425                                     DSQuotaExceededException.class,
1426                                     UnresolvedPathException.class);
1427    }
1428  }
1429
1430  /**
1431   * Rename file or directory.
1432   * @see ClientProtocol#rename(String, String)
1433   * @deprecated Use {@link #rename(String, String, Options.Rename...)} instead.
1434   */
1435  @Deprecated
1436  public boolean rename(String src, String dst) throws IOException {
1437    checkOpen();
1438    try {
1439      return namenode.rename(src, dst);
1440    } catch(RemoteException re) {
1441      throw re.unwrapRemoteException(AccessControlException.class,
1442                                     NSQuotaExceededException.class,
1443                                     DSQuotaExceededException.class,
1444                                     UnresolvedPathException.class);
1445    }
1446  }
1447
1448  /**
1449   * Move blocks from src to trg and delete src
1450   * See {@link ClientProtocol#concat(String, String [])}. 
1451   */
1452  public void concat(String trg, String [] srcs) throws IOException {
1453    checkOpen();
1454    try {
1455      namenode.concat(trg, srcs);
1456    } catch(RemoteException re) {
1457      throw re.unwrapRemoteException(AccessControlException.class,
1458                                     UnresolvedPathException.class);
1459    }
1460  }
1461  /**
1462   * Rename file or directory.
1463   * @see ClientProtocol#rename2(String, String, Options.Rename...)
1464   */
1465  public void rename(String src, String dst, Options.Rename... options)
1466      throws IOException {
1467    checkOpen();
1468    try {
1469      namenode.rename2(src, dst, options);
1470    } catch(RemoteException re) {
1471      throw re.unwrapRemoteException(AccessControlException.class,
1472                                     DSQuotaExceededException.class,
1473                                     FileAlreadyExistsException.class,
1474                                     FileNotFoundException.class,
1475                                     ParentNotDirectoryException.class,
1476                                     SafeModeException.class,
1477                                     NSQuotaExceededException.class,
1478                                     UnresolvedPathException.class);
1479    }
1480  }
1481  /**
1482   * Delete file or directory.
1483   * See {@link ClientProtocol#delete(String, boolean)}. 
1484   */
1485  @Deprecated
1486  public boolean delete(String src) throws IOException {
1487    checkOpen();
1488    return namenode.delete(src, true);
1489  }
1490
1491  /**
1492   * delete file or directory.
1493   * delete contents of the directory if non empty and recursive 
1494   * set to true
1495   *
1496   * @see ClientProtocol#delete(String, boolean)
1497   */
1498  public boolean delete(String src, boolean recursive) throws IOException {
1499    checkOpen();
1500    try {
1501      return namenode.delete(src, recursive);
1502    } catch(RemoteException re) {
1503      throw re.unwrapRemoteException(AccessControlException.class,
1504                                     FileNotFoundException.class,
1505                                     SafeModeException.class,
1506                                     UnresolvedPathException.class);
1507    }
1508  }
1509  
1510  /** Implemented using getFileInfo(src)
1511   */
1512  public boolean exists(String src) throws IOException {
1513    checkOpen();
1514    return getFileInfo(src) != null;
1515  }
1516
1517  /**
1518   * Get a partial listing of the indicated directory
1519   * No block locations need to be fetched
1520   */
1521  public DirectoryListing listPaths(String src,  byte[] startAfter)
1522    throws IOException {
1523    return listPaths(src, startAfter, false);
1524  }
1525  
1526  /**
1527   * Get a partial listing of the indicated directory
1528   *
1529   * Recommend to use HdfsFileStatus.EMPTY_NAME as startAfter
1530   * if the application wants to fetch a listing starting from
1531   * the first entry in the directory
1532   *
1533   * @see ClientProtocol#getListing(String, byte[], boolean)
1534   */
1535  public DirectoryListing listPaths(String src,  byte[] startAfter,
1536      boolean needLocation) 
1537    throws IOException {
1538    checkOpen();
1539    try {
1540      return namenode.getListing(src, startAfter, needLocation);
1541    } catch(RemoteException re) {
1542      throw re.unwrapRemoteException(AccessControlException.class,
1543                                     FileNotFoundException.class,
1544                                     UnresolvedPathException.class);
1545    }
1546  }
1547
1548  /**
1549   * Get the file info for a specific file or directory.
1550   * @param src The string representation of the path to the file
1551   * @return object containing information regarding the file
1552   *         or null if file not found
1553   *         
1554   * @see ClientProtocol#getFileInfo(String) for description of exceptions
1555   */
1556  public HdfsFileStatus getFileInfo(String src) throws IOException {
1557    checkOpen();
1558    try {
1559      return namenode.getFileInfo(src);
1560    } catch(RemoteException re) {
1561      throw re.unwrapRemoteException(AccessControlException.class,
1562                                     FileNotFoundException.class,
1563                                     UnresolvedPathException.class);
1564    }
1565  }
1566
1567  /**
1568   * Get the file info for a specific file or directory. If src
1569   * refers to a symlink then the FileStatus of the link is returned.
1570   * @param src path to a file or directory.
1571   * 
1572   * For description of exceptions thrown 
1573   * @see ClientProtocol#getFileLinkInfo(String)
1574   */
1575  public HdfsFileStatus getFileLinkInfo(String src) throws IOException {
1576    checkOpen();
1577    try {
1578      return namenode.getFileLinkInfo(src);
1579    } catch(RemoteException re) {
1580      throw re.unwrapRemoteException(AccessControlException.class,
1581                                     UnresolvedPathException.class);
1582     }
1583   }
1584
1585  /**
1586   * Get the checksum of a file.
1587   * @param src The file path
1588   * @return The checksum 
1589   * @see DistributedFileSystem#getFileChecksum(Path)
1590   */
1591  public MD5MD5CRC32FileChecksum getFileChecksum(String src) throws IOException {
1592    checkOpen();
1593    return getFileChecksum(src, clientName, namenode, socketFactory,
1594        dfsClientConf.socketTimeout, getDataEncryptionKey(),
1595        dfsClientConf.connectToDnViaHostname);
1596  }
1597  
1598  @InterfaceAudience.Private
1599  public void clearDataEncryptionKey() {
1600    LOG.debug("Clearing encryption key");
1601    synchronized (this) {
1602      encryptionKey = null;
1603    }
1604  }
1605  
1606  /**
1607   * @return true if data sent between this client and DNs should be encrypted,
1608   *         false otherwise.
1609   * @throws IOException in the event of error communicating with the NN
1610   */
1611  boolean shouldEncryptData() throws IOException {
1612    FsServerDefaults d = getServerDefaults();
1613    return d == null ? false : d.getEncryptDataTransfer();
1614  }
1615  
1616  @InterfaceAudience.Private
1617  public DataEncryptionKey getDataEncryptionKey()
1618      throws IOException {
1619    if (shouldEncryptData()) {
1620      synchronized (this) {
1621        if (encryptionKey == null ||
1622            encryptionKey.expiryDate < Time.now()) {
1623          LOG.debug("Getting new encryption token from NN");
1624          encryptionKey = namenode.getDataEncryptionKey();
1625        }
1626        return encryptionKey;
1627      }
1628    } else {
1629      return null;
1630    }
1631  }
1632
1633  /**
1634   * Get the checksum of a file.
1635   * @param src The file path
1636   * @param clientName the name of the client requesting the checksum.
1637   * @param namenode the RPC proxy for the namenode
1638   * @param socketFactory to create sockets to connect to DNs
1639   * @param socketTimeout timeout to use when connecting and waiting for a response
1640   * @param encryptionKey the key needed to communicate with DNs in this cluster
1641   * @param connectToDnViaHostname {@see #connectToDnViaHostname()}
1642   * @return The checksum 
1643   */
1644  static MD5MD5CRC32FileChecksum getFileChecksum(String src,
1645      String clientName,
1646      ClientProtocol namenode, SocketFactory socketFactory, int socketTimeout,
1647      DataEncryptionKey encryptionKey, boolean connectToDnViaHostname)
1648      throws IOException {
1649    //get all block locations
1650    LocatedBlocks blockLocations = callGetBlockLocations(namenode, src, 0, Long.MAX_VALUE);
1651    if (null == blockLocations) {
1652      throw new FileNotFoundException("File does not exist: " + src);
1653    }
1654    List<LocatedBlock> locatedblocks = blockLocations.getLocatedBlocks();
1655    final DataOutputBuffer md5out = new DataOutputBuffer();
1656    int bytesPerCRC = -1;
1657    DataChecksum.Type crcType = DataChecksum.Type.DEFAULT;
1658    long crcPerBlock = 0;
1659    boolean refetchBlocks = false;
1660    int lastRetriedIndex = -1;
1661
1662    //get block checksum for each block
1663    for(int i = 0; i < locatedblocks.size(); i++) {
1664      if (refetchBlocks) {  // refetch to get fresh tokens
1665        blockLocations = callGetBlockLocations(namenode, src, 0, Long.MAX_VALUE);
1666        if (null == blockLocations) {
1667          throw new FileNotFoundException("File does not exist: " + src);
1668        }
1669        locatedblocks = blockLocations.getLocatedBlocks();
1670        refetchBlocks = false;
1671      }
1672      LocatedBlock lb = locatedblocks.get(i);
1673      final ExtendedBlock block = lb.getBlock();
1674      final DatanodeInfo[] datanodes = lb.getLocations();
1675      
1676      //try each datanode location of the block
1677      final int timeout = 3000 * datanodes.length + socketTimeout;
1678      boolean done = false;
1679      for(int j = 0; !done && j < datanodes.length; j++) {
1680        DataOutputStream out = null;
1681        DataInputStream in = null;
1682        
1683        try {
1684          //connect to a datanode
1685          IOStreamPair pair = connectToDN(socketFactory, connectToDnViaHostname,
1686              encryptionKey, datanodes[j], timeout);
1687          out = new DataOutputStream(new BufferedOutputStream(pair.out,
1688              HdfsConstants.SMALL_BUFFER_SIZE));
1689          in = new DataInputStream(pair.in);
1690
1691          if (LOG.isDebugEnabled()) {
1692            LOG.debug("write to " + datanodes[j] + ": "
1693                + Op.BLOCK_CHECKSUM + ", block=" + block);
1694          }
1695          // get block MD5
1696          new Sender(out).blockChecksum(block, lb.getBlockToken());
1697
1698          final BlockOpResponseProto reply =
1699            BlockOpResponseProto.parseFrom(PBHelper.vintPrefixed(in));
1700
1701          if (reply.getStatus() != Status.SUCCESS) {
1702            if (reply.getStatus() == Status.ERROR_ACCESS_TOKEN) {
1703              throw new InvalidBlockTokenException();
1704            } else {
1705              throw new IOException("Bad response " + reply + " for block "
1706                  + block + " from datanode " + datanodes[j]);
1707            }
1708          }
1709          
1710          OpBlockChecksumResponseProto checksumData =
1711            reply.getChecksumResponse();
1712
1713          //read byte-per-checksum
1714          final int bpc = checksumData.getBytesPerCrc();
1715          if (i == 0) { //first block
1716            bytesPerCRC = bpc;
1717          }
1718          else if (bpc != bytesPerCRC) {
1719            throw new IOException("Byte-per-checksum not matched: bpc=" + bpc
1720                + " but bytesPerCRC=" + bytesPerCRC);
1721          }
1722          
1723          //read crc-per-block
1724          final long cpb = checksumData.getCrcPerBlock();
1725          if (locatedblocks.size() > 1 && i == 0) {
1726            crcPerBlock = cpb;
1727          }
1728
1729          //read md5
1730          final MD5Hash md5 = new MD5Hash(
1731              checksumData.getMd5().toByteArray());
1732          md5.write(md5out);
1733          
1734          // read crc-type
1735          final DataChecksum.Type ct;
1736          if (checksumData.hasCrcType()) {
1737            ct = PBHelper.convert(checksumData
1738                .getCrcType());
1739          } else {
1740            LOG.debug("Retrieving checksum from an earlier-version DataNode: " +
1741                      "inferring checksum by reading first byte");
1742            ct = inferChecksumTypeByReading(
1743                clientName, socketFactory, socketTimeout, lb, datanodes[j],
1744                encryptionKey, connectToDnViaHostname);
1745          }
1746
1747          if (i == 0) { // first block
1748            crcType = ct;
1749          } else if (crcType != DataChecksum.Type.MIXED
1750              && crcType != ct) {
1751            // if crc types are mixed in a file
1752            crcType = DataChecksum.Type.MIXED;
1753          }
1754
1755          done = true;
1756
1757          if (LOG.isDebugEnabled()) {
1758            if (i == 0) {
1759              LOG.debug("set bytesPerCRC=" + bytesPerCRC
1760                  + ", crcPerBlock=" + crcPerBlock);
1761            }
1762            LOG.debug("got reply from " + datanodes[j] + ": md5=" + md5);
1763          }
1764        } catch (InvalidBlockTokenException ibte) {
1765          if (i > lastRetriedIndex) {
1766            if (LOG.isDebugEnabled()) {
1767              LOG.debug("Got access token error in response to OP_BLOCK_CHECKSUM "
1768                  + "for file " + src + " for block " + block
1769                  + " from datanode " + datanodes[j]
1770                  + ". Will retry the block once.");
1771            }
1772            lastRetriedIndex = i;
1773            done = true; // actually it's not done; but we'll retry
1774            i--; // repeat at i-th block
1775            refetchBlocks = true;
1776            break;
1777          }
1778        } catch (IOException ie) {
1779          LOG.warn("src=" + src + ", datanodes["+j+"]=" + datanodes[j], ie);
1780        } finally {
1781          IOUtils.closeStream(in);
1782          IOUtils.closeStream(out);
1783        }
1784      }
1785
1786      if (!done) {
1787        throw new IOException("Fail to get block MD5 for " + block);
1788      }
1789    }
1790
1791    //compute file MD5
1792    final MD5Hash fileMD5 = MD5Hash.digest(md5out.getData()); 
1793    switch (crcType) {
1794      case CRC32:
1795        return new MD5MD5CRC32GzipFileChecksum(bytesPerCRC,
1796            crcPerBlock, fileMD5);
1797      case CRC32C:
1798        return new MD5MD5CRC32CastagnoliFileChecksum(bytesPerCRC,
1799            crcPerBlock, fileMD5);
1800      default:
1801        // If there is no block allocated for the file,
1802        // return one with the magic entry that matches what previous
1803        // hdfs versions return.
1804        if (locatedblocks.size() == 0) {
1805          return new MD5MD5CRC32GzipFileChecksum(0, 0, fileMD5);
1806        }
1807
1808        // we should never get here since the validity was checked
1809        // when getCrcType() was called above.
1810        return null;
1811    }
1812  }
1813
1814  /**
1815   * Connect to the given datanode's datantrasfer port, and return
1816   * the resulting IOStreamPair. This includes encryption wrapping, etc.
1817   */
1818  private static IOStreamPair connectToDN(
1819      SocketFactory socketFactory, boolean connectToDnViaHostname,
1820      DataEncryptionKey encryptionKey, DatanodeInfo dn, int timeout)
1821      throws IOException
1822  {
1823    boolean success = false;
1824    Socket sock = null;
1825    try {
1826      sock = socketFactory.createSocket();
1827      String dnAddr = dn.getXferAddr(connectToDnViaHostname);
1828      if (LOG.isDebugEnabled()) {
1829        LOG.debug("Connecting to datanode " + dnAddr);
1830      }
1831      NetUtils.connect(sock, NetUtils.createSocketAddr(dnAddr), timeout);
1832      sock.setSoTimeout(timeout);
1833  
1834      OutputStream unbufOut = NetUtils.getOutputStream(sock);
1835      InputStream unbufIn = NetUtils.getInputStream(sock);
1836      IOStreamPair ret;
1837      if (encryptionKey != null) {
1838        ret = DataTransferEncryptor.getEncryptedStreams(
1839                unbufOut, unbufIn, encryptionKey);
1840      } else {
1841        ret = new IOStreamPair(unbufIn, unbufOut);        
1842      }
1843      success = true;
1844      return ret;
1845    } finally {
1846      if (!success) {
1847        IOUtils.closeSocket(sock);
1848      }
1849    }
1850  }
1851  
1852  /**
1853   * Infer the checksum type for a replica by sending an OP_READ_BLOCK
1854   * for the first byte of that replica. This is used for compatibility
1855   * with older HDFS versions which did not include the checksum type in
1856   * OpBlockChecksumResponseProto.
1857   *
1858   * @param in input stream from datanode
1859   * @param out output stream to datanode
1860   * @param lb the located block
1861   * @param clientName the name of the DFSClient requesting the checksum
1862   * @param dn the connected datanode
1863   * @return the inferred checksum type
1864   * @throws IOException if an error occurs
1865   */
1866  private static Type inferChecksumTypeByReading(
1867      String clientName, SocketFactory socketFactory, int socketTimeout,
1868      LocatedBlock lb, DatanodeInfo dn,
1869      DataEncryptionKey encryptionKey, boolean connectToDnViaHostname)
1870      throws IOException {
1871    IOStreamPair pair = connectToDN(socketFactory, connectToDnViaHostname,
1872        encryptionKey, dn, socketTimeout);
1873
1874    try {
1875      DataOutputStream out = new DataOutputStream(new BufferedOutputStream(pair.out,
1876          HdfsConstants.SMALL_BUFFER_SIZE));
1877      DataInputStream in = new DataInputStream(pair.in);
1878  
1879      new Sender(out).readBlock(lb.getBlock(), lb.getBlockToken(), clientName, 0, 1, true);
1880      final BlockOpResponseProto reply =
1881          BlockOpResponseProto.parseFrom(PBHelper.vintPrefixed(in));
1882      
1883      if (reply.getStatus() != Status.SUCCESS) {
1884        if (reply.getStatus() == Status.ERROR_ACCESS_TOKEN) {
1885          throw new InvalidBlockTokenException();
1886        } else {
1887          throw new IOException("Bad response " + reply + " trying to read "
1888              + lb.getBlock() + " from datanode " + dn);
1889        }
1890      }
1891      
1892      return PBHelper.convert(reply.getReadOpChecksumInfo().getChecksum().getType());
1893    } finally {
1894      IOUtils.cleanup(null, pair.in, pair.out);
1895    }
1896  }
1897
1898  /**
1899   * Set permissions to a file or directory.
1900   * @param src path name.
1901   * @param permission
1902   * 
1903   * @see ClientProtocol#setPermission(String, FsPermission)
1904   */
1905  public void setPermission(String src, FsPermission permission)
1906      throws IOException {
1907    checkOpen();
1908    try {
1909      namenode.setPermission(src, permission);
1910    } catch(RemoteException re) {
1911      throw re.unwrapRemoteException(AccessControlException.class,
1912                                     FileNotFoundException.class,
1913                                     SafeModeException.class,
1914                                     UnresolvedPathException.class);
1915    }
1916  }
1917
1918  /**
1919   * Set file or directory owner.
1920   * @param src path name.
1921   * @param username user id.
1922   * @param groupname user group.
1923   * 
1924   * @see ClientProtocol#setOwner(String, String, String)
1925   */
1926  public void setOwner(String src, String username, String groupname)
1927      throws IOException {
1928    checkOpen();
1929    try {
1930      namenode.setOwner(src, username, groupname);
1931    } catch(RemoteException re) {
1932      throw re.unwrapRemoteException(AccessControlException.class,
1933                                     FileNotFoundException.class,
1934                                     SafeModeException.class,
1935                                     UnresolvedPathException.class);                                   
1936    }
1937  }
1938
1939  /**
1940   * @see ClientProtocol#getStats()
1941   */
1942  public FsStatus getDiskStatus() throws IOException {
1943    long rawNums[] = namenode.getStats();
1944    return new FsStatus(rawNums[0], rawNums[1], rawNums[2]);
1945  }
1946
1947  /**
1948   * Returns count of blocks with no good replicas left. Normally should be 
1949   * zero.
1950   * @throws IOException
1951   */ 
1952  public long getMissingBlocksCount() throws IOException {
1953    return namenode.getStats()[ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX];
1954  }
1955  
1956  /**
1957   * Returns count of blocks with one of more replica missing.
1958   * @throws IOException
1959   */ 
1960  public long getUnderReplicatedBlocksCount() throws IOException {
1961    return namenode.getStats()[ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX];
1962  }
1963  
1964  /**
1965   * Returns count of blocks with at least one replica marked corrupt. 
1966   * @throws IOException
1967   */ 
1968  public long getCorruptBlocksCount() throws IOException {
1969    return namenode.getStats()[ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX];
1970  }
1971  
1972  /**
1973   * @return a list in which each entry describes a corrupt file/block
1974   * @throws IOException
1975   */
1976  public CorruptFileBlocks listCorruptFileBlocks(String path,
1977                                                 String cookie)
1978    throws IOException {
1979    return namenode.listCorruptFileBlocks(path, cookie);
1980  }
1981
1982  public DatanodeInfo[] datanodeReport(DatanodeReportType type)
1983  throws IOException {
1984    return namenode.getDatanodeReport(type);
1985  }
1986    
1987  /**
1988   * Enter, leave or get safe mode.
1989   * 
1990   * @see ClientProtocol#setSafeMode(HdfsConstants.SafeModeAction,boolean)
1991   */
1992  public boolean setSafeMode(SafeModeAction action) throws IOException {
1993    return setSafeMode(action, false);
1994  }
1995  
1996  /**
1997   * Enter, leave or get safe mode.
1998   * 
1999   * @param action
2000   *          One of SafeModeAction.GET, SafeModeAction.ENTER and
2001   *          SafeModeActiob.LEAVE
2002   * @param isChecked
2003   *          If true, then check only active namenode's safemode status, else
2004   *          check first namenode's status.
2005   * @see ClientProtocol#setSafeMode(HdfsConstants.SafeModeAction, boolean)
2006   */
2007  public boolean setSafeMode(SafeModeAction action, boolean isChecked) throws IOException{
2008    return namenode.setSafeMode(action, isChecked);    
2009  }
2010
2011  /**
2012   * Save namespace image.
2013   * 
2014   * @see ClientProtocol#saveNamespace()
2015   */
2016  void saveNamespace() throws AccessControlException, IOException {
2017    try {
2018      namenode.saveNamespace();
2019    } catch(RemoteException re) {
2020      throw re.unwrapRemoteException(AccessControlException.class);
2021    }
2022  }
2023
2024  /**
2025   * Rolls the edit log on the active NameNode.
2026   * @return the txid of the new log segment 
2027   *
2028   * @see ClientProtocol#rollEdits()
2029   */
2030  long rollEdits() throws AccessControlException, IOException {
2031    try {
2032      return namenode.rollEdits();
2033    } catch(RemoteException re) {
2034      throw re.unwrapRemoteException(AccessControlException.class);
2035    }
2036  }
2037  
2038  /**
2039   * enable/disable restore failed storage.
2040   * 
2041   * @see ClientProtocol#restoreFailedStorage(String arg)
2042   */
2043  boolean restoreFailedStorage(String arg)
2044      throws AccessControlException, IOException{
2045    return namenode.restoreFailedStorage(arg);
2046  }
2047
2048  /**
2049   * Refresh the hosts and exclude files.  (Rereads them.)
2050   * See {@link ClientProtocol#refreshNodes()} 
2051   * for more details.
2052   * 
2053   * @see ClientProtocol#refreshNodes()
2054   */
2055  public void refreshNodes() throws IOException {
2056    namenode.refreshNodes();
2057  }
2058
2059  /**
2060   * Dumps DFS data structures into specified file.
2061   * 
2062   * @see ClientProtocol#metaSave(String)
2063   */
2064  public void metaSave(String pathname) throws IOException {
2065    namenode.metaSave(pathname);
2066  }
2067
2068  /**
2069   * Requests the namenode to tell all datanodes to use a new, non-persistent
2070   * bandwidth value for dfs.balance.bandwidthPerSec.
2071   * See {@link ClientProtocol#setBalancerBandwidth(long)} 
2072   * for more details.
2073   * 
2074   * @see ClientProtocol#setBalancerBandwidth(long)
2075   */
2076  public void setBalancerBandwidth(long bandwidth) throws IOException {
2077    namenode.setBalancerBandwidth(bandwidth);
2078  }
2079    
2080  /**
2081   * @see ClientProtocol#finalizeUpgrade()
2082   */
2083  public void finalizeUpgrade() throws IOException {
2084    namenode.finalizeUpgrade();
2085  }
2086
2087  /**
2088   */
2089  @Deprecated
2090  public boolean mkdirs(String src) throws IOException {
2091    return mkdirs(src, null, true);
2092  }
2093
2094  /**
2095   * Create a directory (or hierarchy of directories) with the given
2096   * name and permission.
2097   *
2098   * @param src The path of the directory being created
2099   * @param permission The permission of the directory being created.
2100   * If permission == null, use {@link FsPermission#getDefault()}.
2101   * @param createParent create missing parent directory if true
2102   * 
2103   * @return True if the operation success.
2104   * 
2105   * @see ClientProtocol#mkdirs(String, FsPermission, boolean)
2106   */
2107  public boolean mkdirs(String src, FsPermission permission,
2108      boolean createParent) throws IOException {
2109    if (permission == null) {
2110      permission = FsPermission.getDefault();
2111    }
2112    FsPermission masked = permission.applyUMask(dfsClientConf.uMask);
2113    return primitiveMkdir(src, masked, createParent);
2114  }
2115
2116  /**
2117   * Same {{@link #mkdirs(String, FsPermission, boolean)} except
2118   * that the permissions has already been masked against umask.
2119   */
2120  public boolean primitiveMkdir(String src, FsPermission absPermission)
2121    throws IOException {
2122    return primitiveMkdir(src, absPermission, true);
2123  }
2124
2125  /**
2126   * Same {{@link #mkdirs(String, FsPermission, boolean)} except
2127   * that the permissions has already been masked against umask.
2128   */
2129  public boolean primitiveMkdir(String src, FsPermission absPermission, 
2130    boolean createParent)
2131    throws IOException {
2132    checkOpen();
2133    if (absPermission == null) {
2134      absPermission = 
2135        FsPermission.getDefault().applyUMask(dfsClientConf.uMask);
2136    } 
2137
2138    if(LOG.isDebugEnabled()) {
2139      LOG.debug(src + ": masked=" + absPermission);
2140    }
2141    try {
2142      return namenode.mkdirs(src, absPermission, createParent);
2143    } catch(RemoteException re) {
2144      throw re.unwrapRemoteException(AccessControlException.class,
2145                                     InvalidPathException.class,
2146                                     FileAlreadyExistsException.class,
2147                                     FileNotFoundException.class,
2148                                     ParentNotDirectoryException.class,
2149                                     SafeModeException.class,
2150                                     NSQuotaExceededException.class,
2151                                     DSQuotaExceededException.class,
2152                                     UnresolvedPathException.class);
2153    }
2154  }
2155  
2156  /**
2157   * Get {@link ContentSummary} rooted at the specified directory.
2158   * @param path The string representation of the path
2159   * 
2160   * @see ClientProtocol#getContentSummary(String)
2161   */
2162  ContentSummary getContentSummary(String src) throws IOException {
2163    try {
2164      return namenode.getContentSummary(src);
2165    } catch(RemoteException re) {
2166      throw re.unwrapRemoteException(AccessControlException.class,
2167                                     FileNotFoundException.class,
2168                                     UnresolvedPathException.class);
2169    }
2170  }
2171
2172  /**
2173   * Sets or resets quotas for a directory.
2174   * @see ClientProtocol#setQuota(String, long, long)
2175   */
2176  void setQuota(String src, long namespaceQuota, long diskspaceQuota) 
2177      throws IOException {
2178    // sanity check
2179    if ((namespaceQuota <= 0 && namespaceQuota != HdfsConstants.QUOTA_DONT_SET &&
2180         namespaceQuota != HdfsConstants.QUOTA_RESET) ||
2181        (diskspaceQuota <= 0 && diskspaceQuota != HdfsConstants.QUOTA_DONT_SET &&
2182         diskspaceQuota != HdfsConstants.QUOTA_RESET)) {
2183      throw new IllegalArgumentException("Invalid values for quota : " +
2184                                         namespaceQuota + " and " + 
2185                                         diskspaceQuota);
2186                                         
2187    }
2188    try {
2189      namenode.setQuota(src, namespaceQuota, diskspaceQuota);
2190    } catch(RemoteException re) {
2191      throw re.unwrapRemoteException(AccessControlException.class,
2192                                     FileNotFoundException.class,
2193                                     NSQuotaExceededException.class,
2194                                     DSQuotaExceededException.class,
2195                                     UnresolvedPathException.class);
2196    }
2197  }
2198
2199  /**
2200   * set the modification and access time of a file
2201   * 
2202   * @see ClientProtocol#setTimes(String, long, long)
2203   */
2204  public void setTimes(String src, long mtime, long atime) throws IOException {
2205    checkOpen();
2206    try {
2207      namenode.setTimes(src, mtime, atime);
2208    } catch(RemoteException re) {
2209      throw re.unwrapRemoteException(AccessControlException.class,
2210                                     FileNotFoundException.class,
2211                                     UnresolvedPathException.class);
2212    }
2213  }
2214
2215  /**
2216   * @deprecated use {@link HdfsDataInputStream} instead.
2217   */
2218  @Deprecated
2219  public static class DFSDataInputStream extends HdfsDataInputStream {
2220
2221    public DFSDataInputStream(DFSInputStream in) throws IOException {
2222      super(in);
2223    }
2224  }
2225  
2226  boolean shouldTryShortCircuitRead(InetSocketAddress targetAddr) {
2227    return shortCircuitLocalReads && isLocalAddress(targetAddr);
2228  }
2229
2230  void reportChecksumFailure(String file, ExtendedBlock blk, DatanodeInfo dn) {
2231    DatanodeInfo [] dnArr = { dn };
2232    LocatedBlock [] lblocks = { new LocatedBlock(blk, dnArr) };
2233    reportChecksumFailure(file, lblocks);
2234  }
2235    
2236  // just reports checksum failure and ignores any exception during the report.
2237  void reportChecksumFailure(String file, LocatedBlock lblocks[]) {
2238    try {
2239      reportBadBlocks(lblocks);
2240    } catch (IOException ie) {
2241      LOG.info("Found corruption while reading " + file
2242          + ". Error repairing corrupt blocks. Bad blocks remain.", ie);
2243    }
2244  }
2245
2246  @Override
2247  public String toString() {
2248    return getClass().getSimpleName() + "[clientName=" + clientName
2249        + ", ugi=" + ugi + "]"; 
2250  }
2251
2252  void disableShortCircuit() {
2253    shortCircuitLocalReads = false;
2254  }
2255}