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.fs.CommonConfigurationKeysPublic.HADOOP_SECURITY_CRYPTO_CODEC_CLASSES_KEY_PREFIX;
021import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_SIZE_DEFAULT;
022import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BLOCK_SIZE_KEY;
023import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BYTES_PER_CHECKSUM_DEFAULT;
024import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_BYTES_PER_CHECKSUM_KEY;
025import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_DEFAULT;
026import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_KEY;
027import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_RETRIES_DEFAULT;
028import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_BLOCK_WRITE_RETRIES_KEY;
029import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CACHED_CONN_RETRY_DEFAULT;
030import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CACHED_CONN_RETRY_KEY;
031import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CACHE_DROP_BEHIND_READS;
032import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CACHE_DROP_BEHIND_WRITES;
033import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CACHE_READAHEAD;
034import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CONTEXT;
035import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_CONTEXT_DEFAULT;
036import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_DATANODE_RESTART_TIMEOUT_DEFAULT;
037import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_DATANODE_RESTART_TIMEOUT_KEY;
038import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT;
039import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY;
040import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT;
041import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY;
042import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT;
043import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY;
044import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_DEFAULT;
045import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_KEY;
046import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_READ_PREFETCH_SIZE_KEY;
047import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_RETRY_MAX_ATTEMPTS_DEFAULT;
048import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_RETRY_MAX_ATTEMPTS_KEY;
049import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_RETRY_WINDOW_BASE;
050import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_CAPACITY_DEFAULT;
051import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_CAPACITY_KEY;
052import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_DEFAULT;
053import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_KEY;
054import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_SOCKET_TIMEOUT_KEY;
055import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_USE_DN_HOSTNAME;
056import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_USE_DN_HOSTNAME_DEFAULT;
057import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_WRITE_EXCLUDE_NODES_CACHE_EXPIRY_INTERVAL;
058import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_WRITE_EXCLUDE_NODES_CACHE_EXPIRY_INTERVAL_DEFAULT;
059import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_WRITE_PACKET_SIZE_DEFAULT;
060import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_WRITE_PACKET_SIZE_KEY;
061import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_DATANODE_SOCKET_WRITE_TIMEOUT_KEY;
062import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_REPLICATION_DEFAULT;
063import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_REPLICATION_KEY;
064
065import java.io.BufferedOutputStream;
066import java.io.DataInputStream;
067import java.io.DataOutputStream;
068import java.io.FileNotFoundException;
069import java.io.IOException;
070import java.io.InputStream;
071import java.io.OutputStream;
072import java.lang.reflect.Proxy;
073import java.net.InetAddress;
074import java.net.InetSocketAddress;
075import java.net.Socket;
076import java.net.SocketAddress;
077import java.net.URI;
078import java.net.UnknownHostException;
079import java.nio.charset.Charset;
080import java.security.GeneralSecurityException;
081import java.util.ArrayList;
082import java.util.Collections;
083import java.util.EnumSet;
084import java.util.HashMap;
085import java.util.LinkedHashMap;
086import java.util.List;
087import java.util.Map;
088import java.util.Random;
089import java.util.concurrent.SynchronousQueue;
090import java.util.concurrent.ThreadPoolExecutor;
091import java.util.concurrent.TimeUnit;
092import java.util.concurrent.atomic.AtomicBoolean;
093import java.util.concurrent.atomic.AtomicInteger;
094
095import javax.net.SocketFactory;
096
097import org.apache.commons.logging.Log;
098import org.apache.commons.logging.LogFactory;
099import org.apache.hadoop.HadoopIllegalArgumentException;
100import org.apache.hadoop.classification.InterfaceAudience;
101import org.apache.hadoop.conf.Configuration;
102import org.apache.hadoop.crypto.CipherSuite;
103import org.apache.hadoop.crypto.CryptoCodec;
104import org.apache.hadoop.crypto.CryptoInputStream;
105import org.apache.hadoop.crypto.CryptoOutputStream;
106import org.apache.hadoop.crypto.CryptoProtocolVersion;
107import org.apache.hadoop.crypto.key.KeyProvider;
108import org.apache.hadoop.crypto.key.KeyProvider.KeyVersion;
109import org.apache.hadoop.crypto.key.KeyProviderCryptoExtension;
110import org.apache.hadoop.crypto.key.KeyProviderCryptoExtension.EncryptedKeyVersion;
111import org.apache.hadoop.fs.BlockLocation;
112import org.apache.hadoop.fs.BlockStorageLocation;
113import org.apache.hadoop.fs.CacheFlag;
114import org.apache.hadoop.fs.CommonConfigurationKeysPublic;
115import org.apache.hadoop.fs.ContentSummary;
116import org.apache.hadoop.fs.CreateFlag;
117import org.apache.hadoop.fs.FileAlreadyExistsException;
118import org.apache.hadoop.fs.FileEncryptionInfo;
119import org.apache.hadoop.fs.FileSystem;
120import org.apache.hadoop.fs.FsServerDefaults;
121import org.apache.hadoop.fs.FsStatus;
122import org.apache.hadoop.fs.HdfsBlockLocation;
123import org.apache.hadoop.fs.InvalidPathException;
124import org.apache.hadoop.fs.MD5MD5CRC32CastagnoliFileChecksum;
125import org.apache.hadoop.fs.MD5MD5CRC32FileChecksum;
126import org.apache.hadoop.fs.MD5MD5CRC32GzipFileChecksum;
127import org.apache.hadoop.fs.Options;
128import org.apache.hadoop.fs.Options.ChecksumOpt;
129import org.apache.hadoop.fs.ParentNotDirectoryException;
130import org.apache.hadoop.fs.Path;
131import org.apache.hadoop.fs.StorageType;
132import org.apache.hadoop.fs.RemoteIterator;
133import org.apache.hadoop.fs.UnresolvedLinkException;
134import org.apache.hadoop.fs.VolumeId;
135import org.apache.hadoop.fs.XAttr;
136import org.apache.hadoop.fs.XAttrSetFlag;
137import org.apache.hadoop.fs.permission.AclEntry;
138import org.apache.hadoop.fs.permission.AclStatus;
139import org.apache.hadoop.fs.permission.FsAction;
140import org.apache.hadoop.fs.permission.FsPermission;
141import org.apache.hadoop.hdfs.client.HdfsDataInputStream;
142import org.apache.hadoop.hdfs.client.HdfsDataOutputStream;
143import org.apache.hadoop.hdfs.net.Peer;
144import org.apache.hadoop.hdfs.net.TcpPeerServer;
145import org.apache.hadoop.hdfs.protocol.AclException;
146import org.apache.hadoop.hdfs.protocol.BlockStoragePolicy;
147import org.apache.hadoop.hdfs.protocol.CacheDirectiveEntry;
148import org.apache.hadoop.hdfs.protocol.CacheDirectiveInfo;
149import org.apache.hadoop.hdfs.protocol.CacheDirectiveIterator;
150import org.apache.hadoop.hdfs.protocol.CachePoolEntry;
151import org.apache.hadoop.hdfs.protocol.CachePoolInfo;
152import org.apache.hadoop.hdfs.protocol.CachePoolIterator;
153import org.apache.hadoop.hdfs.protocol.ClientProtocol;
154import org.apache.hadoop.hdfs.protocol.CorruptFileBlocks;
155import org.apache.hadoop.hdfs.protocol.DSQuotaExceededException;
156import org.apache.hadoop.hdfs.protocol.DatanodeID;
157import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
158import org.apache.hadoop.hdfs.protocol.DirectoryListing;
159import org.apache.hadoop.hdfs.protocol.EncryptionZone;
160import org.apache.hadoop.hdfs.protocol.EncryptionZoneIterator;
161import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
162import org.apache.hadoop.hdfs.protocol.HdfsBlocksMetadata;
163import org.apache.hadoop.hdfs.protocol.HdfsConstants;
164import org.apache.hadoop.hdfs.protocol.HdfsConstants.DatanodeReportType;
165import org.apache.hadoop.hdfs.protocol.HdfsConstants.RollingUpgradeAction;
166import org.apache.hadoop.hdfs.protocol.HdfsConstants.SafeModeAction;
167import org.apache.hadoop.hdfs.protocol.HdfsFileStatus;
168import org.apache.hadoop.hdfs.protocol.LastBlockWithStatus;
169import org.apache.hadoop.hdfs.protocol.LocatedBlock;
170import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
171import org.apache.hadoop.hdfs.protocol.NSQuotaExceededException;
172import org.apache.hadoop.hdfs.protocol.QuotaByStorageTypeExceededException;
173import org.apache.hadoop.hdfs.protocol.RollingUpgradeInfo;
174import org.apache.hadoop.hdfs.protocol.SnapshotAccessControlException;
175import org.apache.hadoop.hdfs.protocol.SnapshotDiffReport;
176import org.apache.hadoop.hdfs.protocol.SnapshottableDirectoryStatus;
177import org.apache.hadoop.hdfs.protocol.UnresolvedPathException;
178import org.apache.hadoop.hdfs.protocol.datatransfer.DataTransferProtoUtil;
179import org.apache.hadoop.hdfs.protocol.datatransfer.IOStreamPair;
180import org.apache.hadoop.hdfs.protocol.datatransfer.Op;
181import org.apache.hadoop.hdfs.protocol.datatransfer.ReplaceDatanodeOnFailure;
182import org.apache.hadoop.hdfs.protocol.datatransfer.Sender;
183import org.apache.hadoop.hdfs.protocol.datatransfer.TrustedChannelResolver;
184import org.apache.hadoop.hdfs.protocol.datatransfer.sasl.DataEncryptionKeyFactory;
185import org.apache.hadoop.hdfs.protocol.datatransfer.sasl.DataTransferSaslUtil;
186import org.apache.hadoop.hdfs.protocol.datatransfer.sasl.SaslDataTransferClient;
187import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.BlockOpResponseProto;
188import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.OpBlockChecksumResponseProto;
189import org.apache.hadoop.hdfs.protocol.proto.DataTransferProtos.Status;
190import org.apache.hadoop.hdfs.protocolPB.PBHelper;
191import org.apache.hadoop.hdfs.security.token.block.BlockTokenIdentifier;
192import org.apache.hadoop.hdfs.security.token.block.DataEncryptionKey;
193import org.apache.hadoop.hdfs.security.token.block.InvalidBlockTokenException;
194import org.apache.hadoop.hdfs.security.token.delegation.DelegationTokenIdentifier;
195import org.apache.hadoop.hdfs.server.common.HdfsServerConstants;
196import org.apache.hadoop.hdfs.server.datanode.CachingStrategy;
197import org.apache.hadoop.hdfs.server.namenode.NameNode;
198import org.apache.hadoop.hdfs.server.namenode.SafeModeException;
199import org.apache.hadoop.hdfs.server.protocol.DatanodeStorageReport;
200import org.apache.hadoop.hdfs.util.ByteArrayManager;
201import org.apache.hadoop.io.DataOutputBuffer;
202import org.apache.hadoop.io.EnumSetWritable;
203import org.apache.hadoop.io.IOUtils;
204import org.apache.hadoop.io.MD5Hash;
205import org.apache.hadoop.io.Text;
206import org.apache.hadoop.io.retry.LossyRetryInvocationHandler;
207import org.apache.hadoop.ipc.Client;
208import org.apache.hadoop.ipc.RPC;
209import org.apache.hadoop.ipc.RemoteException;
210import org.apache.hadoop.ipc.RpcInvocationHandler;
211import org.apache.hadoop.net.DNS;
212import org.apache.hadoop.net.NetUtils;
213import org.apache.hadoop.security.AccessControlException;
214import org.apache.hadoop.security.UserGroupInformation;
215import org.apache.hadoop.security.token.SecretManager.InvalidToken;
216import org.apache.hadoop.security.token.Token;
217import org.apache.hadoop.security.token.TokenRenewer;
218import org.apache.hadoop.tracing.SpanReceiverHost;
219import org.apache.hadoop.tracing.TraceUtils;
220import org.apache.hadoop.util.Daemon;
221import org.apache.hadoop.util.DataChecksum;
222import org.apache.hadoop.util.DataChecksum.Type;
223import org.apache.hadoop.util.Progressable;
224import org.apache.hadoop.util.Time;
225import org.apache.htrace.Sampler;
226import org.apache.htrace.SamplerBuilder;
227import org.apache.htrace.Span;
228import org.apache.htrace.Trace;
229import org.apache.htrace.TraceScope;
230
231import com.google.common.annotations.VisibleForTesting;
232import com.google.common.base.Joiner;
233import com.google.common.base.Preconditions;
234import com.google.common.collect.Lists;
235import com.google.common.net.InetAddresses;
236
237/********************************************************
238 * DFSClient can connect to a Hadoop Filesystem and 
239 * perform basic file tasks.  It uses the ClientProtocol
240 * to communicate with a NameNode daemon, and connects 
241 * directly to DataNodes to read/write block data.
242 *
243 * Hadoop DFS users should obtain an instance of 
244 * DistributedFileSystem, which uses DFSClient to handle
245 * filesystem tasks.
246 *
247 ********************************************************/
248@InterfaceAudience.Private
249public class DFSClient implements java.io.Closeable, RemotePeerFactory,
250    DataEncryptionKeyFactory {
251  public static final Log LOG = LogFactory.getLog(DFSClient.class);
252  public static final long SERVER_DEFAULTS_VALIDITY_PERIOD = 60 * 60 * 1000L; // 1 hour
253  static final int TCP_WINDOW_SIZE = 128 * 1024; // 128 KB
254
255  private final Configuration conf;
256  private final Conf dfsClientConf;
257  final ClientProtocol namenode;
258  /* The service used for delegation tokens */
259  private Text dtService;
260
261  final UserGroupInformation ugi;
262  volatile boolean clientRunning = true;
263  volatile long lastLeaseRenewal;
264  private volatile FsServerDefaults serverDefaults;
265  private volatile long serverDefaultsLastUpdate;
266  final String clientName;
267  final SocketFactory socketFactory;
268  final ReplaceDatanodeOnFailure dtpReplaceDatanodeOnFailure;
269  final FileSystem.Statistics stats;
270  private final String authority;
271  private final Random r = new Random();
272  private SocketAddress[] localInterfaceAddrs;
273  private DataEncryptionKey encryptionKey;
274  final SaslDataTransferClient saslClient;
275  private final CachingStrategy defaultReadCachingStrategy;
276  private final CachingStrategy defaultWriteCachingStrategy;
277  private final ClientContext clientContext;
278  private volatile long hedgedReadThresholdMillis;
279  private static final DFSHedgedReadMetrics HEDGED_READ_METRIC =
280      new DFSHedgedReadMetrics();
281  private static ThreadPoolExecutor HEDGED_READ_THREAD_POOL;
282  private final Sampler<?> traceSampler;
283
284  /**
285   * DFSClient configuration 
286   */
287  public static class Conf {
288    final int hdfsTimeout;    // timeout value for a DFS operation.
289
290    final int maxFailoverAttempts;
291    final int maxRetryAttempts;
292    final int failoverSleepBaseMillis;
293    final int failoverSleepMaxMillis;
294    final int maxBlockAcquireFailures;
295    final int confTime;
296    final int ioBufferSize;
297    final ChecksumOpt defaultChecksumOpt;
298    final int writePacketSize;
299    final int writeMaxPackets;
300    final ByteArrayManager.Conf writeByteArrayManagerConf;
301    final int socketTimeout;
302    final int socketCacheCapacity;
303    final long socketCacheExpiry;
304    final long excludedNodesCacheExpiry;
305    /** Wait time window (in msec) if BlockMissingException is caught */
306    final int timeWindow;
307    final int nCachedConnRetry;
308    final int nBlockWriteRetry;
309    final int nBlockWriteLocateFollowingRetry;
310    final long defaultBlockSize;
311    final long prefetchSize;
312    final short defaultReplication;
313    final String taskId;
314    final FsPermission uMask;
315    final boolean connectToDnViaHostname;
316    final boolean getHdfsBlocksMetadataEnabled;
317    final int getFileBlockStorageLocationsNumThreads;
318    final int getFileBlockStorageLocationsTimeoutMs;
319    final int retryTimesForGetLastBlockLength;
320    final int retryIntervalForGetLastBlockLength;
321    final long datanodeRestartTimeout;
322    final long dfsclientSlowIoWarningThresholdMs;
323
324    final boolean useLegacyBlockReader;
325    final boolean useLegacyBlockReaderLocal;
326    final String domainSocketPath;
327    final boolean skipShortCircuitChecksums;
328    final int shortCircuitBufferSize;
329    final boolean shortCircuitLocalReads;
330    final boolean domainSocketDataTraffic;
331    final int shortCircuitStreamsCacheSize;
332    final long shortCircuitStreamsCacheExpiryMs; 
333    final int shortCircuitSharedMemoryWatcherInterruptCheckMs;
334    
335    final boolean shortCircuitMmapEnabled;
336    final int shortCircuitMmapCacheSize;
337    final long shortCircuitMmapCacheExpiryMs;
338    final long shortCircuitMmapCacheRetryTimeout;
339    final long shortCircuitCacheStaleThresholdMs;
340
341    final long keyProviderCacheExpiryMs;
342    public BlockReaderFactory.FailureInjector brfFailureInjector =
343      new BlockReaderFactory.FailureInjector();
344
345    public Conf(Configuration conf) {
346      // The hdfsTimeout is currently the same as the ipc timeout 
347      hdfsTimeout = Client.getTimeout(conf);
348      maxFailoverAttempts = conf.getInt(
349          DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY,
350          DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT);
351      maxRetryAttempts = conf.getInt(
352          DFS_CLIENT_RETRY_MAX_ATTEMPTS_KEY,
353          DFS_CLIENT_RETRY_MAX_ATTEMPTS_DEFAULT);
354      failoverSleepBaseMillis = conf.getInt(
355          DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY,
356          DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT);
357      failoverSleepMaxMillis = conf.getInt(
358          DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY,
359          DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT);
360
361      maxBlockAcquireFailures = conf.getInt(
362          DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_KEY,
363          DFS_CLIENT_MAX_BLOCK_ACQUIRE_FAILURES_DEFAULT);
364      confTime = conf.getInt(DFS_DATANODE_SOCKET_WRITE_TIMEOUT_KEY,
365          HdfsServerConstants.WRITE_TIMEOUT);
366      ioBufferSize = conf.getInt(
367          CommonConfigurationKeysPublic.IO_FILE_BUFFER_SIZE_KEY,
368          CommonConfigurationKeysPublic.IO_FILE_BUFFER_SIZE_DEFAULT);
369      defaultChecksumOpt = getChecksumOptFromConf(conf);
370      socketTimeout = conf.getInt(DFS_CLIENT_SOCKET_TIMEOUT_KEY,
371          HdfsServerConstants.READ_TIMEOUT);
372      /** dfs.write.packet.size is an internal config variable */
373      writePacketSize = conf.getInt(DFS_CLIENT_WRITE_PACKET_SIZE_KEY,
374          DFS_CLIENT_WRITE_PACKET_SIZE_DEFAULT);
375      writeMaxPackets = conf.getInt(
376          DFSConfigKeys.DFS_CLIENT_WRITE_MAX_PACKETS_IN_FLIGHT_KEY,
377          DFSConfigKeys.DFS_CLIENT_WRITE_MAX_PACKETS_IN_FLIGHT_DEFAULT);
378      
379      final boolean byteArrayManagerEnabled = conf.getBoolean(
380          DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_ENABLED_KEY,
381          DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_ENABLED_DEFAULT);
382      if (!byteArrayManagerEnabled) {
383        writeByteArrayManagerConf = null;
384      } else {
385        final int countThreshold = conf.getInt(
386            DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_COUNT_THRESHOLD_KEY,
387            DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_COUNT_THRESHOLD_DEFAULT);
388        final int countLimit = conf.getInt(
389            DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_COUNT_LIMIT_KEY,
390            DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_COUNT_LIMIT_DEFAULT);
391        final long countResetTimePeriodMs = conf.getLong(
392            DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_COUNT_RESET_TIME_PERIOD_MS_KEY,
393            DFSConfigKeys.DFS_CLIENT_WRITE_BYTE_ARRAY_MANAGER_COUNT_RESET_TIME_PERIOD_MS_DEFAULT);
394        writeByteArrayManagerConf = new ByteArrayManager.Conf(
395            countThreshold, countLimit, countResetTimePeriodMs); 
396      }
397      
398      
399      defaultBlockSize = conf.getLongBytes(DFS_BLOCK_SIZE_KEY,
400          DFS_BLOCK_SIZE_DEFAULT);
401      defaultReplication = (short) conf.getInt(
402          DFS_REPLICATION_KEY, DFS_REPLICATION_DEFAULT);
403      taskId = conf.get("mapreduce.task.attempt.id", "NONMAPREDUCE");
404      socketCacheCapacity = conf.getInt(DFS_CLIENT_SOCKET_CACHE_CAPACITY_KEY,
405          DFS_CLIENT_SOCKET_CACHE_CAPACITY_DEFAULT);
406      socketCacheExpiry = conf.getLong(DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_KEY,
407          DFS_CLIENT_SOCKET_CACHE_EXPIRY_MSEC_DEFAULT);
408      excludedNodesCacheExpiry = conf.getLong(
409          DFS_CLIENT_WRITE_EXCLUDE_NODES_CACHE_EXPIRY_INTERVAL,
410          DFS_CLIENT_WRITE_EXCLUDE_NODES_CACHE_EXPIRY_INTERVAL_DEFAULT);
411      prefetchSize = conf.getLong(DFS_CLIENT_READ_PREFETCH_SIZE_KEY,
412          10 * defaultBlockSize);
413      timeWindow = conf.getInt(DFS_CLIENT_RETRY_WINDOW_BASE, 3000);
414      nCachedConnRetry = conf.getInt(DFS_CLIENT_CACHED_CONN_RETRY_KEY,
415          DFS_CLIENT_CACHED_CONN_RETRY_DEFAULT);
416      nBlockWriteRetry = conf.getInt(DFS_CLIENT_BLOCK_WRITE_RETRIES_KEY,
417          DFS_CLIENT_BLOCK_WRITE_RETRIES_DEFAULT);
418      nBlockWriteLocateFollowingRetry = conf.getInt(
419          DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_KEY,
420          DFS_CLIENT_BLOCK_WRITE_LOCATEFOLLOWINGBLOCK_RETRIES_DEFAULT);
421      uMask = FsPermission.getUMask(conf);
422      connectToDnViaHostname = conf.getBoolean(DFS_CLIENT_USE_DN_HOSTNAME,
423          DFS_CLIENT_USE_DN_HOSTNAME_DEFAULT);
424      getHdfsBlocksMetadataEnabled = conf.getBoolean(
425          DFSConfigKeys.DFS_HDFS_BLOCKS_METADATA_ENABLED, 
426          DFSConfigKeys.DFS_HDFS_BLOCKS_METADATA_ENABLED_DEFAULT);
427      getFileBlockStorageLocationsNumThreads = conf.getInt(
428          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_NUM_THREADS,
429          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_NUM_THREADS_DEFAULT);
430      getFileBlockStorageLocationsTimeoutMs = conf.getInt(
431          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_TIMEOUT_MS,
432          DFSConfigKeys.DFS_CLIENT_FILE_BLOCK_STORAGE_LOCATIONS_TIMEOUT_MS_DEFAULT);
433      retryTimesForGetLastBlockLength = conf.getInt(
434          DFSConfigKeys.DFS_CLIENT_RETRY_TIMES_GET_LAST_BLOCK_LENGTH,
435          DFSConfigKeys.DFS_CLIENT_RETRY_TIMES_GET_LAST_BLOCK_LENGTH_DEFAULT);
436      retryIntervalForGetLastBlockLength = conf.getInt(
437        DFSConfigKeys.DFS_CLIENT_RETRY_INTERVAL_GET_LAST_BLOCK_LENGTH,
438        DFSConfigKeys.DFS_CLIENT_RETRY_INTERVAL_GET_LAST_BLOCK_LENGTH_DEFAULT);
439
440      useLegacyBlockReader = conf.getBoolean(
441          DFSConfigKeys.DFS_CLIENT_USE_LEGACY_BLOCKREADER,
442          DFSConfigKeys.DFS_CLIENT_USE_LEGACY_BLOCKREADER_DEFAULT);
443      useLegacyBlockReaderLocal = conf.getBoolean(
444          DFSConfigKeys.DFS_CLIENT_USE_LEGACY_BLOCKREADERLOCAL,
445          DFSConfigKeys.DFS_CLIENT_USE_LEGACY_BLOCKREADERLOCAL_DEFAULT);
446      shortCircuitLocalReads = conf.getBoolean(
447          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_KEY,
448          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_DEFAULT);
449      domainSocketDataTraffic = conf.getBoolean(
450          DFSConfigKeys.DFS_CLIENT_DOMAIN_SOCKET_DATA_TRAFFIC,
451          DFSConfigKeys.DFS_CLIENT_DOMAIN_SOCKET_DATA_TRAFFIC_DEFAULT);
452      domainSocketPath = conf.getTrimmed(
453          DFSConfigKeys.DFS_DOMAIN_SOCKET_PATH_KEY,
454          DFSConfigKeys.DFS_DOMAIN_SOCKET_PATH_DEFAULT);
455
456      if (BlockReaderLocal.LOG.isDebugEnabled()) {
457        BlockReaderLocal.LOG.debug(
458            DFSConfigKeys.DFS_CLIENT_USE_LEGACY_BLOCKREADERLOCAL
459            + " = " + useLegacyBlockReaderLocal);
460        BlockReaderLocal.LOG.debug(
461            DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_KEY
462            + " = " + shortCircuitLocalReads);
463        BlockReaderLocal.LOG.debug(
464            DFSConfigKeys.DFS_CLIENT_DOMAIN_SOCKET_DATA_TRAFFIC
465            + " = " + domainSocketDataTraffic);
466        BlockReaderLocal.LOG.debug(
467            DFSConfigKeys.DFS_DOMAIN_SOCKET_PATH_KEY
468            + " = " + domainSocketPath);
469      }
470
471      skipShortCircuitChecksums = conf.getBoolean(
472          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_SKIP_CHECKSUM_KEY,
473          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_SKIP_CHECKSUM_DEFAULT);
474      shortCircuitBufferSize = conf.getInt(
475          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_BUFFER_SIZE_KEY,
476          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_BUFFER_SIZE_DEFAULT);
477      shortCircuitStreamsCacheSize = conf.getInt(
478          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_STREAMS_CACHE_SIZE_KEY,
479          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_STREAMS_CACHE_SIZE_DEFAULT);
480      shortCircuitStreamsCacheExpiryMs = conf.getLong(
481          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_STREAMS_CACHE_EXPIRY_MS_KEY,
482          DFSConfigKeys.DFS_CLIENT_READ_SHORTCIRCUIT_STREAMS_CACHE_EXPIRY_MS_DEFAULT);
483      shortCircuitMmapEnabled = conf.getBoolean(
484          DFSConfigKeys.DFS_CLIENT_MMAP_ENABLED,
485          DFSConfigKeys.DFS_CLIENT_MMAP_ENABLED_DEFAULT);
486      shortCircuitMmapCacheSize = conf.getInt(
487          DFSConfigKeys.DFS_CLIENT_MMAP_CACHE_SIZE,
488          DFSConfigKeys.DFS_CLIENT_MMAP_CACHE_SIZE_DEFAULT);
489      shortCircuitMmapCacheExpiryMs = conf.getLong(
490          DFSConfigKeys.DFS_CLIENT_MMAP_CACHE_TIMEOUT_MS,
491          DFSConfigKeys.DFS_CLIENT_MMAP_CACHE_TIMEOUT_MS_DEFAULT);
492      shortCircuitMmapCacheRetryTimeout = conf.getLong(
493          DFSConfigKeys.DFS_CLIENT_MMAP_RETRY_TIMEOUT_MS,
494          DFSConfigKeys.DFS_CLIENT_MMAP_RETRY_TIMEOUT_MS_DEFAULT);
495      shortCircuitCacheStaleThresholdMs = conf.getLong(
496          DFSConfigKeys.DFS_CLIENT_SHORT_CIRCUIT_REPLICA_STALE_THRESHOLD_MS,
497          DFSConfigKeys.DFS_CLIENT_SHORT_CIRCUIT_REPLICA_STALE_THRESHOLD_MS_DEFAULT);
498      shortCircuitSharedMemoryWatcherInterruptCheckMs = conf.getInt(
499          DFSConfigKeys.DFS_SHORT_CIRCUIT_SHARED_MEMORY_WATCHER_INTERRUPT_CHECK_MS,
500          DFSConfigKeys.DFS_SHORT_CIRCUIT_SHARED_MEMORY_WATCHER_INTERRUPT_CHECK_MS_DEFAULT);
501
502      datanodeRestartTimeout = conf.getLong(
503          DFS_CLIENT_DATANODE_RESTART_TIMEOUT_KEY,
504          DFS_CLIENT_DATANODE_RESTART_TIMEOUT_DEFAULT) * 1000;
505      dfsclientSlowIoWarningThresholdMs = conf.getLong(
506          DFSConfigKeys.DFS_CLIENT_SLOW_IO_WARNING_THRESHOLD_KEY,
507          DFSConfigKeys.DFS_CLIENT_SLOW_IO_WARNING_THRESHOLD_DEFAULT);
508
509      keyProviderCacheExpiryMs = conf.getLong(
510          DFSConfigKeys.DFS_CLIENT_KEY_PROVIDER_CACHE_EXPIRY_MS,
511          DFSConfigKeys.DFS_CLIENT_KEY_PROVIDER_CACHE_EXPIRY_DEFAULT);
512    }
513
514    public boolean isUseLegacyBlockReaderLocal() {
515      return useLegacyBlockReaderLocal;
516    }
517
518    public String getDomainSocketPath() {
519      return domainSocketPath;
520    }
521
522    public boolean isShortCircuitLocalReads() {
523      return shortCircuitLocalReads;
524    }
525
526    public boolean isDomainSocketDataTraffic() {
527      return domainSocketDataTraffic;
528    }
529
530    private DataChecksum.Type getChecksumType(Configuration conf) {
531      final String checksum = conf.get(
532          DFSConfigKeys.DFS_CHECKSUM_TYPE_KEY,
533          DFSConfigKeys.DFS_CHECKSUM_TYPE_DEFAULT);
534      try {
535        return DataChecksum.Type.valueOf(checksum);
536      } catch(IllegalArgumentException iae) {
537        LOG.warn("Bad checksum type: " + checksum + ". Using default "
538            + DFSConfigKeys.DFS_CHECKSUM_TYPE_DEFAULT);
539        return DataChecksum.Type.valueOf(
540            DFSConfigKeys.DFS_CHECKSUM_TYPE_DEFAULT); 
541      }
542    }
543
544    // Construct a checksum option from conf
545    private ChecksumOpt getChecksumOptFromConf(Configuration conf) {
546      DataChecksum.Type type = getChecksumType(conf);
547      int bytesPerChecksum = conf.getInt(DFS_BYTES_PER_CHECKSUM_KEY,
548          DFS_BYTES_PER_CHECKSUM_DEFAULT);
549      return new ChecksumOpt(type, bytesPerChecksum);
550    }
551
552    // create a DataChecksum with the default option.
553    private DataChecksum createChecksum() throws IOException {
554      return createChecksum(null);
555    }
556
557    private DataChecksum createChecksum(ChecksumOpt userOpt) {
558      // Fill in any missing field with the default.
559      ChecksumOpt myOpt = ChecksumOpt.processChecksumOpt(
560          defaultChecksumOpt, userOpt);
561      DataChecksum dataChecksum = DataChecksum.newDataChecksum(
562          myOpt.getChecksumType(),
563          myOpt.getBytesPerChecksum());
564      if (dataChecksum == null) {
565        throw new HadoopIllegalArgumentException("Invalid checksum type: userOpt="
566            + userOpt + ", default=" + defaultChecksumOpt
567            + ", effective=null");
568      }
569      return dataChecksum;
570    }
571  }
572 
573  public Conf getConf() {
574    return dfsClientConf;
575  }
576
577  Configuration getConfiguration() {
578    return conf;
579  }
580
581  /**
582   * A map from file names to {@link DFSOutputStream} objects
583   * that are currently being written by this client.
584   * Note that a file can only be written by a single client.
585   */
586  private final Map<Long, DFSOutputStream> filesBeingWritten
587      = new HashMap<Long, DFSOutputStream>();
588
589  /**
590   * Same as this(NameNode.getAddress(conf), conf);
591   * @see #DFSClient(InetSocketAddress, Configuration)
592   * @deprecated Deprecated at 0.21
593   */
594  @Deprecated
595  public DFSClient(Configuration conf) throws IOException {
596    this(NameNode.getAddress(conf), conf);
597  }
598  
599  public DFSClient(InetSocketAddress address, Configuration conf) throws IOException {
600    this(NameNode.getUri(address), conf);
601  }
602
603  /**
604   * Same as this(nameNodeUri, conf, null);
605   * @see #DFSClient(URI, Configuration, FileSystem.Statistics)
606   */
607  public DFSClient(URI nameNodeUri, Configuration conf
608      ) throws IOException {
609    this(nameNodeUri, conf, null);
610  }
611
612  /**
613   * Same as this(nameNodeUri, null, conf, stats);
614   * @see #DFSClient(URI, ClientProtocol, Configuration, FileSystem.Statistics) 
615   */
616  public DFSClient(URI nameNodeUri, Configuration conf,
617                   FileSystem.Statistics stats)
618    throws IOException {
619    this(nameNodeUri, null, conf, stats);
620  }
621  
622  /** 
623   * Create a new DFSClient connected to the given nameNodeUri or rpcNamenode.
624   * If HA is enabled and a positive value is set for 
625   * {@link DFSConfigKeys#DFS_CLIENT_TEST_DROP_NAMENODE_RESPONSE_NUM_KEY} in the
626   * configuration, the DFSClient will use {@link LossyRetryInvocationHandler}
627   * as its RetryInvocationHandler. Otherwise one of nameNodeUri or rpcNamenode 
628   * must be null.
629   */
630  @VisibleForTesting
631  public DFSClient(URI nameNodeUri, ClientProtocol rpcNamenode,
632      Configuration conf, FileSystem.Statistics stats)
633    throws IOException {
634    SpanReceiverHost.get(conf, DFSConfigKeys.DFS_CLIENT_HTRACE_PREFIX);
635    traceSampler = new SamplerBuilder(TraceUtils.
636        wrapHadoopConf(DFSConfigKeys.DFS_CLIENT_HTRACE_PREFIX, conf)).build();
637    // Copy only the required DFSClient configuration
638    this.dfsClientConf = new Conf(conf);
639    if (this.dfsClientConf.useLegacyBlockReaderLocal) {
640      LOG.debug("Using legacy short-circuit local reads.");
641    }
642    this.conf = conf;
643    this.stats = stats;
644    this.socketFactory = NetUtils.getSocketFactory(conf, ClientProtocol.class);
645    this.dtpReplaceDatanodeOnFailure = ReplaceDatanodeOnFailure.get(conf);
646
647    this.ugi = UserGroupInformation.getCurrentUser();
648    
649    this.authority = nameNodeUri == null? "null": nameNodeUri.getAuthority();
650    this.clientName = "DFSClient_" + dfsClientConf.taskId + "_" + 
651        DFSUtil.getRandom().nextInt()  + "_" + Thread.currentThread().getId();
652    int numResponseToDrop = conf.getInt(
653        DFSConfigKeys.DFS_CLIENT_TEST_DROP_NAMENODE_RESPONSE_NUM_KEY,
654        DFSConfigKeys.DFS_CLIENT_TEST_DROP_NAMENODE_RESPONSE_NUM_DEFAULT);
655    NameNodeProxies.ProxyAndInfo<ClientProtocol> proxyInfo = null;
656    AtomicBoolean nnFallbackToSimpleAuth = new AtomicBoolean(false);
657    if (numResponseToDrop > 0) {
658      // This case is used for testing.
659      LOG.warn(DFSConfigKeys.DFS_CLIENT_TEST_DROP_NAMENODE_RESPONSE_NUM_KEY
660          + " is set to " + numResponseToDrop
661          + ", this hacked client will proactively drop responses");
662      proxyInfo = NameNodeProxies.createProxyWithLossyRetryHandler(conf,
663          nameNodeUri, ClientProtocol.class, numResponseToDrop,
664          nnFallbackToSimpleAuth);
665    }
666    
667    if (proxyInfo != null) {
668      this.dtService = proxyInfo.getDelegationTokenService();
669      this.namenode = proxyInfo.getProxy();
670    } else if (rpcNamenode != null) {
671      // This case is used for testing.
672      Preconditions.checkArgument(nameNodeUri == null);
673      this.namenode = rpcNamenode;
674      dtService = null;
675    } else {
676      Preconditions.checkArgument(nameNodeUri != null,
677          "null URI");
678      proxyInfo = NameNodeProxies.createProxy(conf, nameNodeUri,
679          ClientProtocol.class, nnFallbackToSimpleAuth);
680      this.dtService = proxyInfo.getDelegationTokenService();
681      this.namenode = proxyInfo.getProxy();
682    }
683
684    String localInterfaces[] =
685      conf.getTrimmedStrings(DFSConfigKeys.DFS_CLIENT_LOCAL_INTERFACES);
686    localInterfaceAddrs = getLocalInterfaceAddrs(localInterfaces);
687    if (LOG.isDebugEnabled() && 0 != localInterfaces.length) {
688      LOG.debug("Using local interfaces [" +
689      Joiner.on(',').join(localInterfaces)+ "] with addresses [" +
690      Joiner.on(',').join(localInterfaceAddrs) + "]");
691    }
692    
693    Boolean readDropBehind = (conf.get(DFS_CLIENT_CACHE_DROP_BEHIND_READS) == null) ?
694        null : conf.getBoolean(DFS_CLIENT_CACHE_DROP_BEHIND_READS, false);
695    Long readahead = (conf.get(DFS_CLIENT_CACHE_READAHEAD) == null) ?
696        null : conf.getLong(DFS_CLIENT_CACHE_READAHEAD, 0);
697    Boolean writeDropBehind = (conf.get(DFS_CLIENT_CACHE_DROP_BEHIND_WRITES) == null) ?
698        null : conf.getBoolean(DFS_CLIENT_CACHE_DROP_BEHIND_WRITES, false);
699    this.defaultReadCachingStrategy =
700        new CachingStrategy(readDropBehind, readahead);
701    this.defaultWriteCachingStrategy =
702        new CachingStrategy(writeDropBehind, readahead);
703    this.clientContext = ClientContext.get(
704        conf.get(DFS_CLIENT_CONTEXT, DFS_CLIENT_CONTEXT_DEFAULT),
705        dfsClientConf);
706    this.hedgedReadThresholdMillis = conf.getLong(
707        DFSConfigKeys.DFS_DFSCLIENT_HEDGED_READ_THRESHOLD_MILLIS,
708        DFSConfigKeys.DEFAULT_DFSCLIENT_HEDGED_READ_THRESHOLD_MILLIS);
709    int numThreads = conf.getInt(
710        DFSConfigKeys.DFS_DFSCLIENT_HEDGED_READ_THREADPOOL_SIZE,
711        DFSConfigKeys.DEFAULT_DFSCLIENT_HEDGED_READ_THREADPOOL_SIZE);
712    if (numThreads > 0) {
713      this.initThreadsNumForHedgedReads(numThreads);
714    }
715    this.saslClient = new SaslDataTransferClient(
716      conf, DataTransferSaslUtil.getSaslPropertiesResolver(conf),
717      TrustedChannelResolver.getInstance(conf), nnFallbackToSimpleAuth);
718  }
719  
720  /**
721   * Return the socket addresses to use with each configured
722   * local interface. Local interfaces may be specified by IP
723   * address, IP address range using CIDR notation, interface
724   * name (e.g. eth0) or sub-interface name (e.g. eth0:0).
725   * The socket addresses consist of the IPs for the interfaces
726   * and the ephemeral port (port 0). If an IP, IP range, or
727   * interface name matches an interface with sub-interfaces
728   * only the IP of the interface is used. Sub-interfaces can
729   * be used by specifying them explicitly (by IP or name).
730   * 
731   * @return SocketAddresses for the configured local interfaces,
732   *    or an empty array if none are configured
733   * @throws UnknownHostException if a given interface name is invalid
734   */
735  private static SocketAddress[] getLocalInterfaceAddrs(
736      String interfaceNames[]) throws UnknownHostException {
737    List<SocketAddress> localAddrs = new ArrayList<SocketAddress>();
738    for (String interfaceName : interfaceNames) {
739      if (InetAddresses.isInetAddress(interfaceName)) {
740        localAddrs.add(new InetSocketAddress(interfaceName, 0));
741      } else if (NetUtils.isValidSubnet(interfaceName)) {
742        for (InetAddress addr : NetUtils.getIPs(interfaceName, false)) {
743          localAddrs.add(new InetSocketAddress(addr, 0));
744        }
745      } else {
746        for (String ip : DNS.getIPs(interfaceName, false)) {
747          localAddrs.add(new InetSocketAddress(ip, 0));
748        }
749      }
750    }
751    return localAddrs.toArray(new SocketAddress[localAddrs.size()]);
752  }
753
754  /**
755   * Select one of the configured local interfaces at random. We use a random
756   * interface because other policies like round-robin are less effective
757   * given that we cache connections to datanodes.
758   *
759   * @return one of the local interface addresses at random, or null if no
760   *    local interfaces are configured
761   */
762  SocketAddress getRandomLocalInterfaceAddr() {
763    if (localInterfaceAddrs.length == 0) {
764      return null;
765    }
766    final int idx = r.nextInt(localInterfaceAddrs.length);
767    final SocketAddress addr = localInterfaceAddrs[idx];
768    if (LOG.isDebugEnabled()) {
769      LOG.debug("Using local interface " + addr);
770    }
771    return addr;
772  }
773
774  /**
775   * Return the number of times the client should go back to the namenode
776   * to retrieve block locations when reading.
777   */
778  int getMaxBlockAcquireFailures() {
779    return dfsClientConf.maxBlockAcquireFailures;
780  }
781
782  /**
783   * Return the timeout that clients should use when writing to datanodes.
784   * @param numNodes the number of nodes in the pipeline.
785   */
786  int getDatanodeWriteTimeout(int numNodes) {
787    return (dfsClientConf.confTime > 0) ?
788      (dfsClientConf.confTime + HdfsServerConstants.WRITE_TIMEOUT_EXTENSION * numNodes) : 0;
789  }
790
791  int getDatanodeReadTimeout(int numNodes) {
792    return dfsClientConf.socketTimeout > 0 ?
793        (HdfsServerConstants.READ_TIMEOUT_EXTENSION * numNodes +
794            dfsClientConf.socketTimeout) : 0;
795  }
796  
797  int getHdfsTimeout() {
798    return dfsClientConf.hdfsTimeout;
799  }
800  
801  @VisibleForTesting
802  public String getClientName() {
803    return clientName;
804  }
805
806  void checkOpen() throws IOException {
807    if (!clientRunning) {
808      IOException result = new IOException("Filesystem closed");
809      throw result;
810    }
811  }
812
813  /** Return the lease renewer instance. The renewer thread won't start
814   *  until the first output stream is created. The same instance will
815   *  be returned until all output streams are closed.
816   */
817  public LeaseRenewer getLeaseRenewer() throws IOException {
818      return LeaseRenewer.getInstance(authority, ugi, this);
819  }
820
821  /** Get a lease and start automatic renewal */
822  private void beginFileLease(final long inodeId, final DFSOutputStream out)
823      throws IOException {
824    getLeaseRenewer().put(inodeId, out, this);
825  }
826
827  /** Stop renewal of lease for the file. */
828  void endFileLease(final long inodeId) throws IOException {
829    getLeaseRenewer().closeFile(inodeId, this);
830  }
831    
832
833  /** Put a file. Only called from LeaseRenewer, where proper locking is
834   *  enforced to consistently update its local dfsclients array and 
835   *  client's filesBeingWritten map.
836   */
837  void putFileBeingWritten(final long inodeId, final DFSOutputStream out) {
838    synchronized(filesBeingWritten) {
839      filesBeingWritten.put(inodeId, out);
840      // update the last lease renewal time only when there was no
841      // writes. once there is one write stream open, the lease renewer
842      // thread keeps it updated well with in anyone's expiration time.
843      if (lastLeaseRenewal == 0) {
844        updateLastLeaseRenewal();
845      }
846    }
847  }
848
849  /** Remove a file. Only called from LeaseRenewer. */
850  void removeFileBeingWritten(final long inodeId) {
851    synchronized(filesBeingWritten) {
852      filesBeingWritten.remove(inodeId);
853      if (filesBeingWritten.isEmpty()) {
854        lastLeaseRenewal = 0;
855      }
856    }
857  }
858
859  /** Is file-being-written map empty? */
860  boolean isFilesBeingWrittenEmpty() {
861    synchronized(filesBeingWritten) {
862      return filesBeingWritten.isEmpty();
863    }
864  }
865  
866  /** @return true if the client is running */
867  boolean isClientRunning() {
868    return clientRunning;
869  }
870
871  long getLastLeaseRenewal() {
872    return lastLeaseRenewal;
873  }
874
875  void updateLastLeaseRenewal() {
876    synchronized(filesBeingWritten) {
877      if (filesBeingWritten.isEmpty()) {
878        return;
879      }
880      lastLeaseRenewal = Time.monotonicNow();
881    }
882  }
883
884  /**
885   * Renew leases.
886   * @return true if lease was renewed. May return false if this
887   * client has been closed or has no files open.
888   **/
889  boolean renewLease() throws IOException {
890    if (clientRunning && !isFilesBeingWrittenEmpty()) {
891      try {
892        namenode.renewLease(clientName);
893        updateLastLeaseRenewal();
894        return true;
895      } catch (IOException e) {
896        // Abort if the lease has already expired. 
897        final long elapsed = Time.monotonicNow() - getLastLeaseRenewal();
898        if (elapsed > HdfsConstants.LEASE_HARDLIMIT_PERIOD) {
899          LOG.warn("Failed to renew lease for " + clientName + " for "
900              + (elapsed/1000) + " seconds (>= hard-limit ="
901              + (HdfsConstants.LEASE_HARDLIMIT_PERIOD/1000) + " seconds.) "
902              + "Closing all files being written ...", e);
903          closeAllFilesBeingWritten(true);
904        } else {
905          // Let the lease renewer handle it and retry.
906          throw e;
907        }
908      }
909    }
910    return false;
911  }
912  
913  /**
914   * Close connections the Namenode.
915   */
916  void closeConnectionToNamenode() {
917    RPC.stopProxy(namenode);
918  }
919  
920  /** Abort and release resources held.  Ignore all errors. */
921  void abort() {
922    clientRunning = false;
923    closeAllFilesBeingWritten(true);
924    try {
925      // remove reference to this client and stop the renewer,
926      // if there is no more clients under the renewer.
927      getLeaseRenewer().closeClient(this);
928    } catch (IOException ioe) {
929       LOG.info("Exception occurred while aborting the client " + ioe);
930    }
931    closeConnectionToNamenode();
932  }
933
934  /** Close/abort all files being written. */
935  private void closeAllFilesBeingWritten(final boolean abort) {
936    for(;;) {
937      final long inodeId;
938      final DFSOutputStream out;
939      synchronized(filesBeingWritten) {
940        if (filesBeingWritten.isEmpty()) {
941          return;
942        }
943        inodeId = filesBeingWritten.keySet().iterator().next();
944        out = filesBeingWritten.remove(inodeId);
945      }
946      if (out != null) {
947        try {
948          if (abort) {
949            out.abort();
950          } else {
951            out.close();
952          }
953        } catch(IOException ie) {
954          LOG.error("Failed to " + (abort? "abort": "close") +
955                  " inode " + inodeId, ie);
956        }
957      }
958    }
959  }
960
961  /**
962   * Close the file system, abandoning all of the leases and files being
963   * created and close connections to the namenode.
964   */
965  @Override
966  public synchronized void close() throws IOException {
967    if(clientRunning) {
968      closeAllFilesBeingWritten(false);
969      clientRunning = false;
970      getLeaseRenewer().closeClient(this);
971      // close connections to the namenode
972      closeConnectionToNamenode();
973    }
974  }
975
976  /**
977   * Close all open streams, abandoning all of the leases and files being
978   * created.
979   * @param abort whether streams should be gracefully closed
980   */
981  public void closeOutputStreams(boolean abort) {
982    if (clientRunning) {
983      closeAllFilesBeingWritten(abort);
984    }
985  }
986
987  /**
988   * Get the default block size for this cluster
989   * @return the default block size in bytes
990   */
991  public long getDefaultBlockSize() {
992    return dfsClientConf.defaultBlockSize;
993  }
994    
995  /**
996   * @see ClientProtocol#getPreferredBlockSize(String)
997   */
998  public long getBlockSize(String f) throws IOException {
999    TraceScope scope = getPathTraceScope("getBlockSize", f);
1000    try {
1001      return namenode.getPreferredBlockSize(f);
1002    } catch (IOException ie) {
1003      LOG.warn("Problem getting block size", ie);
1004      throw ie;
1005    } finally {
1006      scope.close();
1007    }
1008  }
1009
1010  /**
1011   * Get server default values for a number of configuration params.
1012   * @see ClientProtocol#getServerDefaults()
1013   */
1014  public FsServerDefaults getServerDefaults() throws IOException {
1015    long now = Time.monotonicNow();
1016    if ((serverDefaults == null) ||
1017        (now - serverDefaultsLastUpdate > SERVER_DEFAULTS_VALIDITY_PERIOD)) {
1018      serverDefaults = namenode.getServerDefaults();
1019      serverDefaultsLastUpdate = now;
1020    }
1021    assert serverDefaults != null;
1022    return serverDefaults;
1023  }
1024  
1025  /**
1026   * Get a canonical token service name for this client's tokens.  Null should
1027   * be returned if the client is not using tokens.
1028   * @return the token service for the client
1029   */
1030  @InterfaceAudience.LimitedPrivate( { "HDFS" }) 
1031  public String getCanonicalServiceName() {
1032    return (dtService != null) ? dtService.toString() : null;
1033  }
1034  
1035  /**
1036   * @see ClientProtocol#getDelegationToken(Text)
1037   */
1038  public Token<DelegationTokenIdentifier> getDelegationToken(Text renewer)
1039      throws IOException {
1040    assert dtService != null;
1041    TraceScope scope = Trace.startSpan("getDelegationToken", traceSampler);
1042    try {
1043      Token<DelegationTokenIdentifier> token =
1044        namenode.getDelegationToken(renewer);
1045      if (token != null) {
1046        token.setService(this.dtService);
1047        LOG.info("Created " + DelegationTokenIdentifier.stringifyToken(token));
1048      } else {
1049        LOG.info("Cannot get delegation token from " + renewer);
1050      }
1051      return token;
1052    } finally {
1053      scope.close();
1054    }
1055  }
1056
1057  /**
1058   * Renew a delegation token
1059   * @param token the token to renew
1060   * @return the new expiration time
1061   * @throws InvalidToken
1062   * @throws IOException
1063   * @deprecated Use Token.renew instead.
1064   */
1065  @Deprecated
1066  public long renewDelegationToken(Token<DelegationTokenIdentifier> token)
1067      throws InvalidToken, IOException {
1068    LOG.info("Renewing " + DelegationTokenIdentifier.stringifyToken(token));
1069    try {
1070      return token.renew(conf);
1071    } catch (InterruptedException ie) {                                       
1072      throw new RuntimeException("caught interrupted", ie);
1073    } catch (RemoteException re) {
1074      throw re.unwrapRemoteException(InvalidToken.class,
1075                                     AccessControlException.class);
1076    }
1077  }
1078  
1079  private static final Map<String, Boolean> localAddrMap = Collections
1080      .synchronizedMap(new HashMap<String, Boolean>());
1081  
1082  public static boolean isLocalAddress(InetSocketAddress targetAddr) {
1083    InetAddress addr = targetAddr.getAddress();
1084    Boolean cached = localAddrMap.get(addr.getHostAddress());
1085    if (cached != null) {
1086      if (LOG.isTraceEnabled()) {
1087        LOG.trace("Address " + targetAddr +
1088                  (cached ? " is local" : " is not local"));
1089      }
1090      return cached;
1091    }
1092    
1093    boolean local = NetUtils.isLocalAddress(addr);
1094
1095    if (LOG.isTraceEnabled()) {
1096      LOG.trace("Address " + targetAddr +
1097                (local ? " is local" : " is not local"));
1098    }
1099    localAddrMap.put(addr.getHostAddress(), local);
1100    return local;
1101  }
1102  
1103  /**
1104   * Cancel a delegation token
1105   * @param token the token to cancel
1106   * @throws InvalidToken
1107   * @throws IOException
1108   * @deprecated Use Token.cancel instead.
1109   */
1110  @Deprecated
1111  public void cancelDelegationToken(Token<DelegationTokenIdentifier> token)
1112      throws InvalidToken, IOException {
1113    LOG.info("Cancelling " + DelegationTokenIdentifier.stringifyToken(token));
1114    try {
1115      token.cancel(conf);
1116     } catch (InterruptedException ie) {                                       
1117      throw new RuntimeException("caught interrupted", ie);
1118    } catch (RemoteException re) {
1119      throw re.unwrapRemoteException(InvalidToken.class,
1120                                     AccessControlException.class);
1121    }
1122  }
1123  
1124  @InterfaceAudience.Private
1125  public static class Renewer extends TokenRenewer {
1126    
1127    static {
1128      //Ensure that HDFS Configuration files are loaded before trying to use
1129      // the renewer.
1130      HdfsConfiguration.init();
1131    }
1132    
1133    @Override
1134    public boolean handleKind(Text kind) {
1135      return DelegationTokenIdentifier.HDFS_DELEGATION_KIND.equals(kind);
1136    }
1137
1138    @SuppressWarnings("unchecked")
1139    @Override
1140    public long renew(Token<?> token, Configuration conf) throws IOException {
1141      Token<DelegationTokenIdentifier> delToken = 
1142        (Token<DelegationTokenIdentifier>) token;
1143      ClientProtocol nn = getNNProxy(delToken, conf);
1144      try {
1145        return nn.renewDelegationToken(delToken);
1146      } catch (RemoteException re) {
1147        throw re.unwrapRemoteException(InvalidToken.class, 
1148                                       AccessControlException.class);
1149      }
1150    }
1151
1152    @SuppressWarnings("unchecked")
1153    @Override
1154    public void cancel(Token<?> token, Configuration conf) throws IOException {
1155      Token<DelegationTokenIdentifier> delToken = 
1156          (Token<DelegationTokenIdentifier>) token;
1157      LOG.info("Cancelling " + 
1158               DelegationTokenIdentifier.stringifyToken(delToken));
1159      ClientProtocol nn = getNNProxy(delToken, conf);
1160      try {
1161        nn.cancelDelegationToken(delToken);
1162      } catch (RemoteException re) {
1163        throw re.unwrapRemoteException(InvalidToken.class,
1164            AccessControlException.class);
1165      }
1166    }
1167    
1168    private static ClientProtocol getNNProxy(
1169        Token<DelegationTokenIdentifier> token, Configuration conf)
1170        throws IOException {
1171      URI uri = HAUtil.getServiceUriFromToken(HdfsConstants.HDFS_URI_SCHEME,
1172              token);
1173      if (HAUtil.isTokenForLogicalUri(token) &&
1174          !HAUtil.isLogicalUri(conf, uri)) {
1175        // If the token is for a logical nameservice, but the configuration
1176        // we have disagrees about that, we can't actually renew it.
1177        // This can be the case in MR, for example, if the RM doesn't
1178        // have all of the HA clusters configured in its configuration.
1179        throw new IOException("Unable to map logical nameservice URI '" +
1180            uri + "' to a NameNode. Local configuration does not have " +
1181            "a failover proxy provider configured.");
1182      }
1183      
1184      NameNodeProxies.ProxyAndInfo<ClientProtocol> info =
1185        NameNodeProxies.createProxy(conf, uri, ClientProtocol.class);
1186      assert info.getDelegationTokenService().equals(token.getService()) :
1187        "Returned service '" + info.getDelegationTokenService().toString() +
1188        "' doesn't match expected service '" +
1189        token.getService().toString() + "'";
1190        
1191      return info.getProxy();
1192    }
1193
1194    @Override
1195    public boolean isManaged(Token<?> token) throws IOException {
1196      return true;
1197    }
1198    
1199  }
1200
1201  /**
1202   * Report corrupt blocks that were discovered by the client.
1203   * @see ClientProtocol#reportBadBlocks(LocatedBlock[])
1204   */
1205  public void reportBadBlocks(LocatedBlock[] blocks) throws IOException {
1206    namenode.reportBadBlocks(blocks);
1207  }
1208  
1209  public short getDefaultReplication() {
1210    return dfsClientConf.defaultReplication;
1211  }
1212  
1213  public LocatedBlocks getLocatedBlocks(String src, long start)
1214      throws IOException {
1215    return getLocatedBlocks(src, start, dfsClientConf.prefetchSize);
1216  }
1217
1218  /*
1219   * This is just a wrapper around callGetBlockLocations, but non-static so that
1220   * we can stub it out for tests.
1221   */
1222  @VisibleForTesting
1223  public LocatedBlocks getLocatedBlocks(String src, long start, long length)
1224      throws IOException {
1225    TraceScope scope = getPathTraceScope("getBlockLocations", src);
1226    try {
1227      return callGetBlockLocations(namenode, src, start, length);
1228    } finally {
1229      scope.close();
1230    }
1231  }
1232
1233  /**
1234   * @see ClientProtocol#getBlockLocations(String, long, long)
1235   */
1236  static LocatedBlocks callGetBlockLocations(ClientProtocol namenode,
1237      String src, long start, long length) 
1238      throws IOException {
1239    try {
1240      return namenode.getBlockLocations(src, start, length);
1241    } catch(RemoteException re) {
1242      throw re.unwrapRemoteException(AccessControlException.class,
1243                                     FileNotFoundException.class,
1244                                     UnresolvedPathException.class);
1245    }
1246  }
1247
1248  /**
1249   * Recover a file's lease
1250   * @param src a file's path
1251   * @return true if the file is already closed
1252   * @throws IOException
1253   */
1254  boolean recoverLease(String src) throws IOException {
1255    checkOpen();
1256
1257    TraceScope scope = getPathTraceScope("recoverLease", src);
1258    try {
1259      return namenode.recoverLease(src, clientName);
1260    } catch (RemoteException re) {
1261      throw re.unwrapRemoteException(FileNotFoundException.class,
1262                                     AccessControlException.class,
1263                                     UnresolvedPathException.class);
1264    } finally {
1265      scope.close();
1266    }
1267  }
1268
1269  /**
1270   * Get block location info about file
1271   * 
1272   * getBlockLocations() returns a list of hostnames that store 
1273   * data for a specific file region.  It returns a set of hostnames
1274   * for every block within the indicated region.
1275   *
1276   * This function is very useful when writing code that considers
1277   * data-placement when performing operations.  For example, the
1278   * MapReduce system tries to schedule tasks on the same machines
1279   * as the data-block the task processes. 
1280   */
1281  public BlockLocation[] getBlockLocations(String src, long start, 
1282        long length) throws IOException, UnresolvedLinkException {
1283    TraceScope scope = getPathTraceScope("getBlockLocations", src);
1284    try {
1285      LocatedBlocks blocks = getLocatedBlocks(src, start, length);
1286      BlockLocation[] locations =  DFSUtil.locatedBlocks2Locations(blocks);
1287      HdfsBlockLocation[] hdfsLocations = new HdfsBlockLocation[locations.length];
1288      for (int i = 0; i < locations.length; i++) {
1289        hdfsLocations[i] = new HdfsBlockLocation(locations[i], blocks.get(i));
1290      }
1291      return hdfsLocations;
1292    } finally {
1293      scope.close();
1294    }
1295  }
1296  
1297  /**
1298   * Get block location information about a list of {@link HdfsBlockLocation}.
1299   * Used by {@link DistributedFileSystem#getFileBlockStorageLocations(List)} to
1300   * get {@link BlockStorageLocation}s for blocks returned by
1301   * {@link DistributedFileSystem#getFileBlockLocations(org.apache.hadoop.fs.FileStatus, long, long)}
1302   * .
1303   * 
1304   * This is done by making a round of RPCs to the associated datanodes, asking
1305   * the volume of each block replica. The returned array of
1306   * {@link BlockStorageLocation} expose this information as a
1307   * {@link VolumeId}.
1308   * 
1309   * @param blockLocations
1310   *          target blocks on which to query volume location information
1311   * @return volumeBlockLocations original block array augmented with additional
1312   *         volume location information for each replica.
1313   */
1314  public BlockStorageLocation[] getBlockStorageLocations(
1315      List<BlockLocation> blockLocations) throws IOException,
1316      UnsupportedOperationException, InvalidBlockTokenException {
1317    if (!getConf().getHdfsBlocksMetadataEnabled) {
1318      throw new UnsupportedOperationException("Datanode-side support for " +
1319          "getVolumeBlockLocations() must also be enabled in the client " +
1320          "configuration.");
1321    }
1322    // Downcast blockLocations and fetch out required LocatedBlock(s)
1323    List<LocatedBlock> blocks = new ArrayList<LocatedBlock>();
1324    for (BlockLocation loc : blockLocations) {
1325      if (!(loc instanceof HdfsBlockLocation)) {
1326        throw new ClassCastException("DFSClient#getVolumeBlockLocations " +
1327            "expected to be passed HdfsBlockLocations");
1328      }
1329      HdfsBlockLocation hdfsLoc = (HdfsBlockLocation) loc;
1330      blocks.add(hdfsLoc.getLocatedBlock());
1331    }
1332    
1333    // Re-group the LocatedBlocks to be grouped by datanodes, with the values
1334    // a list of the LocatedBlocks on the datanode.
1335    Map<DatanodeInfo, List<LocatedBlock>> datanodeBlocks = 
1336        new LinkedHashMap<DatanodeInfo, List<LocatedBlock>>();
1337    for (LocatedBlock b : blocks) {
1338      for (DatanodeInfo info : b.getLocations()) {
1339        if (!datanodeBlocks.containsKey(info)) {
1340          datanodeBlocks.put(info, new ArrayList<LocatedBlock>());
1341        }
1342        List<LocatedBlock> l = datanodeBlocks.get(info);
1343        l.add(b);
1344      }
1345    }
1346        
1347    // Make RPCs to the datanodes to get volume locations for its replicas
1348    TraceScope scope =
1349      Trace.startSpan("getBlockStorageLocations", traceSampler);
1350    Map<DatanodeInfo, HdfsBlocksMetadata> metadatas;
1351    try {
1352      metadatas = BlockStorageLocationUtil.
1353          queryDatanodesForHdfsBlocksMetadata(conf, datanodeBlocks,
1354              getConf().getFileBlockStorageLocationsNumThreads,
1355              getConf().getFileBlockStorageLocationsTimeoutMs,
1356              getConf().connectToDnViaHostname);
1357      if (LOG.isTraceEnabled()) {
1358        LOG.trace("metadata returned: "
1359            + Joiner.on("\n").withKeyValueSeparator("=").join(metadatas));
1360      }
1361    } finally {
1362      scope.close();
1363    }
1364    
1365    // Regroup the returned VolumeId metadata to again be grouped by
1366    // LocatedBlock rather than by datanode
1367    Map<LocatedBlock, List<VolumeId>> blockVolumeIds = BlockStorageLocationUtil
1368        .associateVolumeIdsWithBlocks(blocks, metadatas);
1369    
1370    // Combine original BlockLocations with new VolumeId information
1371    BlockStorageLocation[] volumeBlockLocations = BlockStorageLocationUtil
1372        .convertToVolumeBlockLocations(blocks, blockVolumeIds);
1373
1374    return volumeBlockLocations;
1375  }
1376
1377  /**
1378   * Decrypts a EDEK by consulting the KeyProvider.
1379   */
1380  private KeyVersion decryptEncryptedDataEncryptionKey(FileEncryptionInfo
1381      feInfo) throws IOException {
1382    TraceScope scope = Trace.startSpan("decryptEDEK", traceSampler);
1383    try {
1384      KeyProvider provider = getKeyProvider();
1385      if (provider == null) {
1386        throw new IOException("No KeyProvider is configured, cannot access" +
1387            " an encrypted file");
1388      }
1389      EncryptedKeyVersion ekv = EncryptedKeyVersion.createForDecryption(
1390          feInfo.getKeyName(), feInfo.getEzKeyVersionName(), feInfo.getIV(),
1391          feInfo.getEncryptedDataEncryptionKey());
1392      try {
1393        KeyProviderCryptoExtension cryptoProvider = KeyProviderCryptoExtension
1394            .createKeyProviderCryptoExtension(provider);
1395        return cryptoProvider.decryptEncryptedKey(ekv);
1396      } catch (GeneralSecurityException e) {
1397        throw new IOException(e);
1398      }
1399    } finally {
1400      scope.close();
1401    }
1402  }
1403
1404  /**
1405   * Obtain the crypto protocol version from the provided FileEncryptionInfo,
1406   * checking to see if this version is supported by.
1407   *
1408   * @param feInfo FileEncryptionInfo
1409   * @return CryptoProtocolVersion from the feInfo
1410   * @throws IOException if the protocol version is unsupported.
1411   */
1412  private static CryptoProtocolVersion getCryptoProtocolVersion
1413      (FileEncryptionInfo feInfo) throws IOException {
1414    final CryptoProtocolVersion version = feInfo.getCryptoProtocolVersion();
1415    if (!CryptoProtocolVersion.supports(version)) {
1416      throw new IOException("Client does not support specified " +
1417          "CryptoProtocolVersion " + version.getDescription() + " version " +
1418          "number" + version.getVersion());
1419    }
1420    return version;
1421  }
1422
1423  /**
1424   * Obtain a CryptoCodec based on the CipherSuite set in a FileEncryptionInfo
1425   * and the available CryptoCodecs configured in the Configuration.
1426   *
1427   * @param conf   Configuration
1428   * @param feInfo FileEncryptionInfo
1429   * @return CryptoCodec
1430   * @throws IOException if no suitable CryptoCodec for the CipherSuite is
1431   *                     available.
1432   */
1433  private static CryptoCodec getCryptoCodec(Configuration conf,
1434      FileEncryptionInfo feInfo) throws IOException {
1435    final CipherSuite suite = feInfo.getCipherSuite();
1436    if (suite.equals(CipherSuite.UNKNOWN)) {
1437      throw new IOException("NameNode specified unknown CipherSuite with ID "
1438          + suite.getUnknownValue() + ", cannot instantiate CryptoCodec.");
1439    }
1440    final CryptoCodec codec = CryptoCodec.getInstance(conf, suite);
1441    if (codec == null) {
1442      throw new UnknownCipherSuiteException(
1443          "No configuration found for the cipher suite "
1444          + suite.getConfigSuffix() + " prefixed with "
1445          + HADOOP_SECURITY_CRYPTO_CODEC_CLASSES_KEY_PREFIX
1446          + ". Please see the example configuration "
1447          + "hadoop.security.crypto.codec.classes.EXAMPLECIPHERSUITE "
1448          + "at core-default.xml for details.");
1449    }
1450    return codec;
1451  }
1452
1453  /**
1454   * Wraps the stream in a CryptoInputStream if the underlying file is
1455   * encrypted.
1456   */
1457  public HdfsDataInputStream createWrappedInputStream(DFSInputStream dfsis)
1458      throws IOException {
1459    final FileEncryptionInfo feInfo = dfsis.getFileEncryptionInfo();
1460    if (feInfo != null) {
1461      // File is encrypted, wrap the stream in a crypto stream.
1462      // Currently only one version, so no special logic based on the version #
1463      getCryptoProtocolVersion(feInfo);
1464      final CryptoCodec codec = getCryptoCodec(conf, feInfo);
1465      final KeyVersion decrypted = decryptEncryptedDataEncryptionKey(feInfo);
1466      final CryptoInputStream cryptoIn =
1467          new CryptoInputStream(dfsis, codec, decrypted.getMaterial(),
1468              feInfo.getIV());
1469      return new HdfsDataInputStream(cryptoIn);
1470    } else {
1471      // No FileEncryptionInfo so no encryption.
1472      return new HdfsDataInputStream(dfsis);
1473    }
1474  }
1475
1476  /**
1477   * Wraps the stream in a CryptoOutputStream if the underlying file is
1478   * encrypted.
1479   */
1480  public HdfsDataOutputStream createWrappedOutputStream(DFSOutputStream dfsos,
1481      FileSystem.Statistics statistics) throws IOException {
1482    return createWrappedOutputStream(dfsos, statistics, 0);
1483  }
1484
1485  /**
1486   * Wraps the stream in a CryptoOutputStream if the underlying file is
1487   * encrypted.
1488   */
1489  public HdfsDataOutputStream createWrappedOutputStream(DFSOutputStream dfsos,
1490      FileSystem.Statistics statistics, long startPos) throws IOException {
1491    final FileEncryptionInfo feInfo = dfsos.getFileEncryptionInfo();
1492    if (feInfo != null) {
1493      // File is encrypted, wrap the stream in a crypto stream.
1494      // Currently only one version, so no special logic based on the version #
1495      getCryptoProtocolVersion(feInfo);
1496      final CryptoCodec codec = getCryptoCodec(conf, feInfo);
1497      KeyVersion decrypted = decryptEncryptedDataEncryptionKey(feInfo);
1498      final CryptoOutputStream cryptoOut =
1499          new CryptoOutputStream(dfsos, codec,
1500              decrypted.getMaterial(), feInfo.getIV(), startPos);
1501      return new HdfsDataOutputStream(cryptoOut, statistics, startPos);
1502    } else {
1503      // No FileEncryptionInfo present so no encryption.
1504      return new HdfsDataOutputStream(dfsos, statistics, startPos);
1505    }
1506  }
1507
1508  public DFSInputStream open(String src) 
1509      throws IOException, UnresolvedLinkException {
1510    return open(src, dfsClientConf.ioBufferSize, true, null);
1511  }
1512
1513  /**
1514   * Create an input stream that obtains a nodelist from the
1515   * namenode, and then reads from all the right places.  Creates
1516   * inner subclass of InputStream that does the right out-of-band
1517   * work.
1518   * @deprecated Use {@link #open(String, int, boolean)} instead.
1519   */
1520  @Deprecated
1521  public DFSInputStream open(String src, int buffersize, boolean verifyChecksum,
1522                             FileSystem.Statistics stats)
1523      throws IOException, UnresolvedLinkException {
1524    return open(src, buffersize, verifyChecksum);
1525  }
1526  
1527
1528  /**
1529   * Create an input stream that obtains a nodelist from the
1530   * namenode, and then reads from all the right places.  Creates
1531   * inner subclass of InputStream that does the right out-of-band
1532   * work.
1533   */
1534  public DFSInputStream open(String src, int buffersize, boolean verifyChecksum)
1535      throws IOException, UnresolvedLinkException {
1536    checkOpen();
1537    //    Get block info from namenode
1538    TraceScope scope = getPathTraceScope("newDFSInputStream", src);
1539    try {
1540      return new DFSInputStream(this, src, verifyChecksum);
1541    } finally {
1542      scope.close();
1543    }
1544  }
1545
1546  /**
1547   * Get the namenode associated with this DFSClient object
1548   * @return the namenode associated with this DFSClient object
1549   */
1550  public ClientProtocol getNamenode() {
1551    return namenode;
1552  }
1553  
1554  /**
1555   * Call {@link #create(String, boolean, short, long, Progressable)} with
1556   * default <code>replication</code> and <code>blockSize<code> and null <code>
1557   * progress</code>.
1558   */
1559  public OutputStream create(String src, boolean overwrite) 
1560      throws IOException {
1561    return create(src, overwrite, dfsClientConf.defaultReplication,
1562        dfsClientConf.defaultBlockSize, null);
1563  }
1564    
1565  /**
1566   * Call {@link #create(String, boolean, short, long, Progressable)} with
1567   * default <code>replication</code> and <code>blockSize<code>.
1568   */
1569  public OutputStream create(String src, 
1570                             boolean overwrite,
1571                             Progressable progress) throws IOException {
1572    return create(src, overwrite, dfsClientConf.defaultReplication,
1573        dfsClientConf.defaultBlockSize, progress);
1574  }
1575    
1576  /**
1577   * Call {@link #create(String, boolean, short, long, Progressable)} with
1578   * null <code>progress</code>.
1579   */
1580  public OutputStream create(String src, 
1581                             boolean overwrite, 
1582                             short replication,
1583                             long blockSize) throws IOException {
1584    return create(src, overwrite, replication, blockSize, null);
1585  }
1586
1587  /**
1588   * Call {@link #create(String, boolean, short, long, Progressable, int)}
1589   * with default bufferSize.
1590   */
1591  public OutputStream create(String src, boolean overwrite, short replication,
1592      long blockSize, Progressable progress) throws IOException {
1593    return create(src, overwrite, replication, blockSize, progress,
1594        dfsClientConf.ioBufferSize);
1595  }
1596
1597  /**
1598   * Call {@link #create(String, FsPermission, EnumSet, short, long, 
1599   * Progressable, int, ChecksumOpt)} with default <code>permission</code>
1600   * {@link FsPermission#getFileDefault()}.
1601   * 
1602   * @param src File name
1603   * @param overwrite overwrite an existing file if true
1604   * @param replication replication factor for the file
1605   * @param blockSize maximum block size
1606   * @param progress interface for reporting client progress
1607   * @param buffersize underlying buffersize
1608   * 
1609   * @return output stream
1610   */
1611  public OutputStream create(String src,
1612                             boolean overwrite,
1613                             short replication,
1614                             long blockSize,
1615                             Progressable progress,
1616                             int buffersize)
1617      throws IOException {
1618    return create(src, FsPermission.getFileDefault(),
1619        overwrite ? EnumSet.of(CreateFlag.CREATE, CreateFlag.OVERWRITE)
1620            : EnumSet.of(CreateFlag.CREATE), replication, blockSize, progress,
1621        buffersize, null);
1622  }
1623
1624  /**
1625   * Call {@link #create(String, FsPermission, EnumSet, boolean, short, 
1626   * long, Progressable, int, ChecksumOpt)} with <code>createParent</code>
1627   *  set to true.
1628   */
1629  public DFSOutputStream create(String src, 
1630                             FsPermission permission,
1631                             EnumSet<CreateFlag> flag, 
1632                             short replication,
1633                             long blockSize,
1634                             Progressable progress,
1635                             int buffersize,
1636                             ChecksumOpt checksumOpt)
1637      throws IOException {
1638    return create(src, permission, flag, true,
1639        replication, blockSize, progress, buffersize, checksumOpt, null);
1640  }
1641
1642  /**
1643   * Create a new dfs file with the specified block replication 
1644   * with write-progress reporting and return an output stream for writing
1645   * into the file.  
1646   * 
1647   * @param src File name
1648   * @param permission The permission of the directory being created.
1649   *          If null, use default permission {@link FsPermission#getFileDefault()}
1650   * @param flag indicates create a new file or create/overwrite an
1651   *          existing file or append to an existing file
1652   * @param createParent create missing parent directory if true
1653   * @param replication block replication
1654   * @param blockSize maximum block size
1655   * @param progress interface for reporting client progress
1656   * @param buffersize underlying buffer size 
1657   * @param checksumOpt checksum options
1658   * 
1659   * @return output stream
1660   *
1661   * @see ClientProtocol#create for detailed description of exceptions thrown
1662   */
1663  public DFSOutputStream create(String src, 
1664                             FsPermission permission,
1665                             EnumSet<CreateFlag> flag, 
1666                             boolean createParent,
1667                             short replication,
1668                             long blockSize,
1669                             Progressable progress,
1670                             int buffersize,
1671                             ChecksumOpt checksumOpt) throws IOException {
1672    return create(src, permission, flag, createParent, replication, blockSize, 
1673        progress, buffersize, checksumOpt, null);
1674  }
1675
1676  /**
1677   * Same as {@link #create(String, FsPermission, EnumSet, boolean, short, long,
1678   * Progressable, int, ChecksumOpt)} with the addition of favoredNodes that is
1679   * a hint to where the namenode should place the file blocks.
1680   * The favored nodes hint is not persisted in HDFS. Hence it may be honored
1681   * at the creation time only. HDFS could move the blocks during balancing or
1682   * replication, to move the blocks from favored nodes. A value of null means
1683   * no favored nodes for this create
1684   */
1685  public DFSOutputStream create(String src, 
1686                             FsPermission permission,
1687                             EnumSet<CreateFlag> flag, 
1688                             boolean createParent,
1689                             short replication,
1690                             long blockSize,
1691                             Progressable progress,
1692                             int buffersize,
1693                             ChecksumOpt checksumOpt,
1694                             InetSocketAddress[] favoredNodes) throws IOException {
1695    checkOpen();
1696    if (permission == null) {
1697      permission = FsPermission.getFileDefault();
1698    }
1699    FsPermission masked = permission.applyUMask(dfsClientConf.uMask);
1700    if(LOG.isDebugEnabled()) {
1701      LOG.debug(src + ": masked=" + masked);
1702    }
1703    final DFSOutputStream result = DFSOutputStream.newStreamForCreate(this,
1704        src, masked, flag, createParent, replication, blockSize, progress,
1705        buffersize, dfsClientConf.createChecksum(checksumOpt),
1706        getFavoredNodesStr(favoredNodes));
1707    beginFileLease(result.getFileId(), result);
1708    return result;
1709  }
1710
1711  private String[] getFavoredNodesStr(InetSocketAddress[] favoredNodes) {
1712    String[] favoredNodeStrs = null;
1713    if (favoredNodes != null) {
1714      favoredNodeStrs = new String[favoredNodes.length];
1715      for (int i = 0; i < favoredNodes.length; i++) {
1716        favoredNodeStrs[i] = 
1717            favoredNodes[i].getHostName() + ":" 
1718                         + favoredNodes[i].getPort();
1719      }
1720    }
1721    return favoredNodeStrs;
1722  }
1723  
1724  /**
1725   * Append to an existing file if {@link CreateFlag#APPEND} is present
1726   */
1727  private DFSOutputStream primitiveAppend(String src, EnumSet<CreateFlag> flag,
1728      int buffersize, Progressable progress) throws IOException {
1729    if (flag.contains(CreateFlag.APPEND)) {
1730      HdfsFileStatus stat = getFileInfo(src);
1731      if (stat == null) { // No file to append to
1732        // New file needs to be created if create option is present
1733        if (!flag.contains(CreateFlag.CREATE)) {
1734          throw new FileNotFoundException("failed to append to non-existent file "
1735              + src + " on client " + clientName);
1736        }
1737        return null;
1738      }
1739      return callAppend(src, buffersize, flag, progress, null);
1740    }
1741    return null;
1742  }
1743  
1744  /**
1745   * Same as {{@link #create(String, FsPermission, EnumSet, short, long,
1746   *  Progressable, int, ChecksumOpt)} except that the permission
1747   *  is absolute (ie has already been masked with umask.
1748   */
1749  public DFSOutputStream primitiveCreate(String src, 
1750                             FsPermission absPermission,
1751                             EnumSet<CreateFlag> flag,
1752                             boolean createParent,
1753                             short replication,
1754                             long blockSize,
1755                             Progressable progress,
1756                             int buffersize,
1757                             ChecksumOpt checksumOpt)
1758      throws IOException, UnresolvedLinkException {
1759    checkOpen();
1760    CreateFlag.validate(flag);
1761    DFSOutputStream result = primitiveAppend(src, flag, buffersize, progress);
1762    if (result == null) {
1763      DataChecksum checksum = dfsClientConf.createChecksum(checksumOpt);
1764      result = DFSOutputStream.newStreamForCreate(this, src, absPermission,
1765          flag, createParent, replication, blockSize, progress, buffersize,
1766          checksum, null);
1767    }
1768    beginFileLease(result.getFileId(), result);
1769    return result;
1770  }
1771  
1772  /**
1773   * Creates a symbolic link.
1774   * 
1775   * @see ClientProtocol#createSymlink(String, String,FsPermission, boolean) 
1776   */
1777  public void createSymlink(String target, String link, boolean createParent)
1778      throws IOException {
1779    TraceScope scope = getPathTraceScope("createSymlink", target);
1780    try {
1781      FsPermission dirPerm = 
1782          FsPermission.getDefault().applyUMask(dfsClientConf.uMask); 
1783      namenode.createSymlink(target, link, dirPerm, createParent);
1784    } catch (RemoteException re) {
1785      throw re.unwrapRemoteException(AccessControlException.class,
1786                                     FileAlreadyExistsException.class, 
1787                                     FileNotFoundException.class,
1788                                     ParentNotDirectoryException.class,
1789                                     NSQuotaExceededException.class, 
1790                                     DSQuotaExceededException.class,
1791                                     UnresolvedPathException.class,
1792                                     SnapshotAccessControlException.class);
1793    } finally {
1794      scope.close();
1795    }
1796  }
1797
1798  /**
1799   * Resolve the *first* symlink, if any, in the path.
1800   * 
1801   * @see ClientProtocol#getLinkTarget(String)
1802   */
1803  public String getLinkTarget(String path) throws IOException { 
1804    checkOpen();
1805    TraceScope scope = getPathTraceScope("getLinkTarget", path);
1806    try {
1807      return namenode.getLinkTarget(path);
1808    } catch (RemoteException re) {
1809      throw re.unwrapRemoteException(AccessControlException.class,
1810                                     FileNotFoundException.class);
1811    } finally {
1812      scope.close();
1813    }
1814  }
1815
1816  /** Method to get stream returned by append call */
1817  private DFSOutputStream callAppend(String src, int buffersize,
1818      EnumSet<CreateFlag> flag, Progressable progress, String[] favoredNodes)
1819      throws IOException {
1820    CreateFlag.validateForAppend(flag);
1821    try {
1822      LastBlockWithStatus blkWithStatus = namenode.append(src, clientName,
1823          new EnumSetWritable<>(flag, CreateFlag.class));
1824      return DFSOutputStream.newStreamForAppend(this, src, flag, buffersize,
1825          progress, blkWithStatus.getLastBlock(),
1826          blkWithStatus.getFileStatus(), dfsClientConf.createChecksum(),
1827          favoredNodes);
1828    } catch(RemoteException re) {
1829      throw re.unwrapRemoteException(AccessControlException.class,
1830                                     FileNotFoundException.class,
1831                                     SafeModeException.class,
1832                                     DSQuotaExceededException.class,
1833                                     UnsupportedOperationException.class,
1834                                     UnresolvedPathException.class,
1835                                     SnapshotAccessControlException.class);
1836    }
1837  }
1838  
1839  /**
1840   * Append to an existing HDFS file.  
1841   * 
1842   * @param src file name
1843   * @param buffersize buffer size
1844   * @param flag indicates whether to append data to a new block instead of
1845   *             the last block
1846   * @param progress for reporting write-progress; null is acceptable.
1847   * @param statistics file system statistics; null is acceptable.
1848   * @return an output stream for writing into the file
1849   * 
1850   * @see ClientProtocol#append(String, String, EnumSetWritable)
1851   */
1852  public HdfsDataOutputStream append(final String src, final int buffersize,
1853      EnumSet<CreateFlag> flag, final Progressable progress,
1854      final FileSystem.Statistics statistics) throws IOException {
1855    final DFSOutputStream out = append(src, buffersize, flag, null, progress);
1856    return createWrappedOutputStream(out, statistics, out.getInitialLen());
1857  }
1858
1859  /**
1860   * Append to an existing HDFS file.
1861   * 
1862   * @param src file name
1863   * @param buffersize buffer size
1864   * @param flag indicates whether to append data to a new block instead of the
1865   *          last block
1866   * @param progress for reporting write-progress; null is acceptable.
1867   * @param statistics file system statistics; null is acceptable.
1868   * @param favoredNodes FavoredNodes for new blocks
1869   * @return an output stream for writing into the file
1870   * @see ClientProtocol#append(String, String, EnumSetWritable)
1871   */
1872  public HdfsDataOutputStream append(final String src, final int buffersize,
1873      EnumSet<CreateFlag> flag, final Progressable progress,
1874      final FileSystem.Statistics statistics,
1875      final InetSocketAddress[] favoredNodes) throws IOException {
1876    final DFSOutputStream out = append(src, buffersize, flag,
1877        getFavoredNodesStr(favoredNodes), progress);
1878    return createWrappedOutputStream(out, statistics, out.getInitialLen());
1879  }
1880
1881  private DFSOutputStream append(String src, int buffersize,
1882      EnumSet<CreateFlag> flag, String[] favoredNodes, Progressable progress)
1883      throws IOException {
1884    checkOpen();
1885    final DFSOutputStream result = callAppend(src, buffersize, flag, progress,
1886        favoredNodes);
1887    beginFileLease(result.getFileId(), result);
1888    return result;
1889  }
1890
1891  /**
1892   * Set replication for an existing file.
1893   * @param src file name
1894   * @param replication replication to set the file to
1895   * 
1896   * @see ClientProtocol#setReplication(String, short)
1897   */
1898  public boolean setReplication(String src, short replication)
1899      throws IOException {
1900    TraceScope scope = getPathTraceScope("setReplication", src);
1901    try {
1902      return namenode.setReplication(src, replication);
1903    } catch(RemoteException re) {
1904      throw re.unwrapRemoteException(AccessControlException.class,
1905                                     FileNotFoundException.class,
1906                                     SafeModeException.class,
1907                                     DSQuotaExceededException.class,
1908                                     UnresolvedPathException.class,
1909                                     SnapshotAccessControlException.class);
1910    } finally {
1911      scope.close();
1912    }
1913  }
1914
1915  /**
1916   * Set storage policy for an existing file/directory
1917   * @param src file/directory name
1918   * @param policyName name of the storage policy
1919   */
1920  public void setStoragePolicy(String src, String policyName)
1921      throws IOException {
1922    TraceScope scope = getPathTraceScope("setStoragePolicy", src);
1923    try {
1924      namenode.setStoragePolicy(src, policyName);
1925    } catch (RemoteException e) {
1926      throw e.unwrapRemoteException(AccessControlException.class,
1927                                    FileNotFoundException.class,
1928                                    SafeModeException.class,
1929                                    NSQuotaExceededException.class,
1930                                    UnresolvedPathException.class,
1931                                    SnapshotAccessControlException.class);
1932    } finally {
1933      scope.close();
1934    }
1935  }
1936
1937  /**
1938   * @return All the existing storage policies
1939   */
1940  public BlockStoragePolicy[] getStoragePolicies() throws IOException {
1941    TraceScope scope = Trace.startSpan("getStoragePolicies", traceSampler);
1942    try {
1943      return namenode.getStoragePolicies();
1944    } finally {
1945      scope.close();
1946    }
1947  }
1948
1949  /**
1950   * Rename file or directory.
1951   * @see ClientProtocol#rename(String, String)
1952   * @deprecated Use {@link #rename(String, String, Options.Rename...)} instead.
1953   */
1954  @Deprecated
1955  public boolean rename(String src, String dst) throws IOException {
1956    checkOpen();
1957    TraceScope scope = getSrcDstTraceScope("rename", src, dst);
1958    try {
1959      return namenode.rename(src, dst);
1960    } catch(RemoteException re) {
1961      throw re.unwrapRemoteException(AccessControlException.class,
1962                                     NSQuotaExceededException.class,
1963                                     DSQuotaExceededException.class,
1964                                     UnresolvedPathException.class,
1965                                     SnapshotAccessControlException.class);
1966    } finally {
1967      scope.close();
1968    }
1969  }
1970
1971  /**
1972   * Move blocks from src to trg and delete src
1973   * See {@link ClientProtocol#concat}.
1974   */
1975  public void concat(String trg, String [] srcs) throws IOException {
1976    checkOpen();
1977    TraceScope scope = Trace.startSpan("concat", traceSampler);
1978    try {
1979      namenode.concat(trg, srcs);
1980    } catch(RemoteException re) {
1981      throw re.unwrapRemoteException(AccessControlException.class,
1982                                     UnresolvedPathException.class,
1983                                     SnapshotAccessControlException.class);
1984    } finally {
1985      scope.close();
1986    }
1987  }
1988  /**
1989   * Rename file or directory.
1990   * @see ClientProtocol#rename2(String, String, Options.Rename...)
1991   */
1992  public void rename(String src, String dst, Options.Rename... options)
1993      throws IOException {
1994    checkOpen();
1995    TraceScope scope = getSrcDstTraceScope("rename2", src, dst);
1996    try {
1997      namenode.rename2(src, dst, options);
1998    } catch(RemoteException re) {
1999      throw re.unwrapRemoteException(AccessControlException.class,
2000                                     DSQuotaExceededException.class,
2001                                     FileAlreadyExistsException.class,
2002                                     FileNotFoundException.class,
2003                                     ParentNotDirectoryException.class,
2004                                     SafeModeException.class,
2005                                     NSQuotaExceededException.class,
2006                                     UnresolvedPathException.class,
2007                                     SnapshotAccessControlException.class);
2008    } finally {
2009      scope.close();
2010    }
2011  }
2012
2013  /**
2014   * Truncate a file to an indicated size
2015   * See {@link ClientProtocol#truncate}.
2016   */
2017  public boolean truncate(String src, long newLength) throws IOException {
2018    checkOpen();
2019    if (newLength < 0) {
2020      throw new HadoopIllegalArgumentException(
2021          "Cannot truncate to a negative file size: " + newLength + ".");
2022    }
2023    try {
2024      return namenode.truncate(src, newLength, clientName);
2025    } catch (RemoteException re) {
2026      throw re.unwrapRemoteException(AccessControlException.class,
2027          UnresolvedPathException.class);
2028    }
2029  }
2030
2031  /**
2032   * Delete file or directory.
2033   * See {@link ClientProtocol#delete(String, boolean)}. 
2034   */
2035  @Deprecated
2036  public boolean delete(String src) throws IOException {
2037    checkOpen();
2038    return delete(src, true);
2039  }
2040
2041  /**
2042   * delete file or directory.
2043   * delete contents of the directory if non empty and recursive 
2044   * set to true
2045   *
2046   * @see ClientProtocol#delete(String, boolean)
2047   */
2048  public boolean delete(String src, boolean recursive) throws IOException {
2049    checkOpen();
2050    TraceScope scope = getPathTraceScope("delete", src);
2051    try {
2052      return namenode.delete(src, recursive);
2053    } catch(RemoteException re) {
2054      throw re.unwrapRemoteException(AccessControlException.class,
2055                                     FileNotFoundException.class,
2056                                     SafeModeException.class,
2057                                     UnresolvedPathException.class,
2058                                     SnapshotAccessControlException.class);
2059    } finally {
2060      scope.close();
2061    }
2062  }
2063  
2064  /** Implemented using getFileInfo(src)
2065   */
2066  public boolean exists(String src) throws IOException {
2067    checkOpen();
2068    return getFileInfo(src) != null;
2069  }
2070
2071  /**
2072   * Get a partial listing of the indicated directory
2073   * No block locations need to be fetched
2074   */
2075  public DirectoryListing listPaths(String src,  byte[] startAfter)
2076    throws IOException {
2077    return listPaths(src, startAfter, false);
2078  }
2079  
2080  /**
2081   * Get a partial listing of the indicated directory
2082   *
2083   * Recommend to use HdfsFileStatus.EMPTY_NAME as startAfter
2084   * if the application wants to fetch a listing starting from
2085   * the first entry in the directory
2086   *
2087   * @see ClientProtocol#getListing(String, byte[], boolean)
2088   */
2089  public DirectoryListing listPaths(String src,  byte[] startAfter,
2090      boolean needLocation) throws IOException {
2091    checkOpen();
2092    TraceScope scope = getPathTraceScope("listPaths", src);
2093    try {
2094      return namenode.getListing(src, startAfter, needLocation);
2095    } catch(RemoteException re) {
2096      throw re.unwrapRemoteException(AccessControlException.class,
2097                                     FileNotFoundException.class,
2098                                     UnresolvedPathException.class);
2099    } finally {
2100      scope.close();
2101    }
2102  }
2103
2104  /**
2105   * Get the file info for a specific file or directory.
2106   * @param src The string representation of the path to the file
2107   * @return object containing information regarding the file
2108   *         or null if file not found
2109   *         
2110   * @see ClientProtocol#getFileInfo(String) for description of exceptions
2111   */
2112  public HdfsFileStatus getFileInfo(String src) throws IOException {
2113    checkOpen();
2114    TraceScope scope = getPathTraceScope("getFileInfo", src);
2115    try {
2116      return namenode.getFileInfo(src);
2117    } catch(RemoteException re) {
2118      throw re.unwrapRemoteException(AccessControlException.class,
2119                                     FileNotFoundException.class,
2120                                     UnresolvedPathException.class);
2121    } finally {
2122      scope.close();
2123    }
2124  }
2125  
2126  /**
2127   * Close status of a file
2128   * @return true if file is already closed
2129   */
2130  public boolean isFileClosed(String src) throws IOException{
2131    checkOpen();
2132    TraceScope scope = getPathTraceScope("isFileClosed", src);
2133    try {
2134      return namenode.isFileClosed(src);
2135    } catch(RemoteException re) {
2136      throw re.unwrapRemoteException(AccessControlException.class,
2137                                     FileNotFoundException.class,
2138                                     UnresolvedPathException.class);
2139    } finally {
2140      scope.close();
2141    }
2142  }
2143  
2144  /**
2145   * Get the file info for a specific file or directory. If src
2146   * refers to a symlink then the FileStatus of the link is returned.
2147   * @param src path to a file or directory.
2148   * 
2149   * For description of exceptions thrown 
2150   * @see ClientProtocol#getFileLinkInfo(String)
2151   */
2152  public HdfsFileStatus getFileLinkInfo(String src) throws IOException {
2153    checkOpen();
2154    TraceScope scope = getPathTraceScope("getFileLinkInfo", src);
2155    try {
2156      return namenode.getFileLinkInfo(src);
2157    } catch(RemoteException re) {
2158      throw re.unwrapRemoteException(AccessControlException.class,
2159                                     UnresolvedPathException.class);
2160    } finally {
2161      scope.close();
2162    }
2163   }
2164  
2165  @InterfaceAudience.Private
2166  public void clearDataEncryptionKey() {
2167    LOG.debug("Clearing encryption key");
2168    synchronized (this) {
2169      encryptionKey = null;
2170    }
2171  }
2172  
2173  /**
2174   * @return true if data sent between this client and DNs should be encrypted,
2175   *         false otherwise.
2176   * @throws IOException in the event of error communicating with the NN
2177   */
2178  boolean shouldEncryptData() throws IOException {
2179    FsServerDefaults d = getServerDefaults();
2180    return d == null ? false : d.getEncryptDataTransfer();
2181  }
2182  
2183  @Override
2184  public DataEncryptionKey newDataEncryptionKey() throws IOException {
2185    if (shouldEncryptData()) {
2186      synchronized (this) {
2187        if (encryptionKey == null ||
2188            encryptionKey.expiryDate < Time.now()) {
2189          LOG.debug("Getting new encryption token from NN");
2190          encryptionKey = namenode.getDataEncryptionKey();
2191        }
2192        return encryptionKey;
2193      }
2194    } else {
2195      return null;
2196    }
2197  }
2198
2199  /**
2200   * Get the checksum of the whole file of a range of the file. Note that the
2201   * range always starts from the beginning of the file.
2202   * @param src The file path
2203   * @param length the length of the range, i.e., the range is [0, length]
2204   * @return The checksum 
2205   * @see DistributedFileSystem#getFileChecksum(Path)
2206   */
2207  public MD5MD5CRC32FileChecksum getFileChecksum(String src, long length)
2208      throws IOException {
2209    checkOpen();
2210    Preconditions.checkArgument(length >= 0);
2211    //get block locations for the file range
2212    LocatedBlocks blockLocations = callGetBlockLocations(namenode, src, 0,
2213        length);
2214    if (null == blockLocations) {
2215      throw new FileNotFoundException("File does not exist: " + src);
2216    }
2217    List<LocatedBlock> locatedblocks = blockLocations.getLocatedBlocks();
2218    final DataOutputBuffer md5out = new DataOutputBuffer();
2219    int bytesPerCRC = -1;
2220    DataChecksum.Type crcType = DataChecksum.Type.DEFAULT;
2221    long crcPerBlock = 0;
2222    boolean refetchBlocks = false;
2223    int lastRetriedIndex = -1;
2224
2225    // get block checksum for each block
2226    long remaining = length;
2227    if (src.contains(HdfsConstants.SEPARATOR_DOT_SNAPSHOT_DIR_SEPARATOR)) {
2228      remaining = Math.min(length, blockLocations.getFileLength());
2229    }
2230    for(int i = 0; i < locatedblocks.size() && remaining > 0; i++) {
2231      if (refetchBlocks) {  // refetch to get fresh tokens
2232        blockLocations = callGetBlockLocations(namenode, src, 0, length);
2233        if (null == blockLocations) {
2234          throw new FileNotFoundException("File does not exist: " + src);
2235        }
2236        locatedblocks = blockLocations.getLocatedBlocks();
2237        refetchBlocks = false;
2238      }
2239      LocatedBlock lb = locatedblocks.get(i);
2240      final ExtendedBlock block = lb.getBlock();
2241      if (remaining < block.getNumBytes()) {
2242        block.setNumBytes(remaining);
2243      }
2244      remaining -= block.getNumBytes();
2245      final DatanodeInfo[] datanodes = lb.getLocations();
2246      
2247      //try each datanode location of the block
2248      final int timeout = 3000 * datanodes.length + dfsClientConf.socketTimeout;
2249      boolean done = false;
2250      for(int j = 0; !done && j < datanodes.length; j++) {
2251        DataOutputStream out = null;
2252        DataInputStream in = null;
2253        
2254        try {
2255          //connect to a datanode
2256          IOStreamPair pair = connectToDN(datanodes[j], timeout, lb);
2257          out = new DataOutputStream(new BufferedOutputStream(pair.out,
2258              HdfsConstants.SMALL_BUFFER_SIZE));
2259          in = new DataInputStream(pair.in);
2260
2261          if (LOG.isDebugEnabled()) {
2262            LOG.debug("write to " + datanodes[j] + ": "
2263                + Op.BLOCK_CHECKSUM + ", block=" + block);
2264          }
2265          // get block MD5
2266          new Sender(out).blockChecksum(block, lb.getBlockToken());
2267
2268          final BlockOpResponseProto reply =
2269            BlockOpResponseProto.parseFrom(PBHelper.vintPrefixed(in));
2270
2271          String logInfo = "for block " + block + " from datanode " + datanodes[j];
2272          DataTransferProtoUtil.checkBlockOpStatus(reply, logInfo);
2273
2274          OpBlockChecksumResponseProto checksumData =
2275            reply.getChecksumResponse();
2276
2277          //read byte-per-checksum
2278          final int bpc = checksumData.getBytesPerCrc();
2279          if (i == 0) { //first block
2280            bytesPerCRC = bpc;
2281          }
2282          else if (bpc != bytesPerCRC) {
2283            throw new IOException("Byte-per-checksum not matched: bpc=" + bpc
2284                + " but bytesPerCRC=" + bytesPerCRC);
2285          }
2286          
2287          //read crc-per-block
2288          final long cpb = checksumData.getCrcPerBlock();
2289          if (locatedblocks.size() > 1 && i == 0) {
2290            crcPerBlock = cpb;
2291          }
2292
2293          //read md5
2294          final MD5Hash md5 = new MD5Hash(
2295              checksumData.getMd5().toByteArray());
2296          md5.write(md5out);
2297          
2298          // read crc-type
2299          final DataChecksum.Type ct;
2300          if (checksumData.hasCrcType()) {
2301            ct = PBHelper.convert(checksumData
2302                .getCrcType());
2303          } else {
2304            LOG.debug("Retrieving checksum from an earlier-version DataNode: " +
2305                      "inferring checksum by reading first byte");
2306            ct = inferChecksumTypeByReading(lb, datanodes[j]);
2307          }
2308
2309          if (i == 0) { // first block
2310            crcType = ct;
2311          } else if (crcType != DataChecksum.Type.MIXED
2312              && crcType != ct) {
2313            // if crc types are mixed in a file
2314            crcType = DataChecksum.Type.MIXED;
2315          }
2316
2317          done = true;
2318
2319          if (LOG.isDebugEnabled()) {
2320            if (i == 0) {
2321              LOG.debug("set bytesPerCRC=" + bytesPerCRC
2322                  + ", crcPerBlock=" + crcPerBlock);
2323            }
2324            LOG.debug("got reply from " + datanodes[j] + ": md5=" + md5);
2325          }
2326        } catch (InvalidBlockTokenException ibte) {
2327          if (i > lastRetriedIndex) {
2328            if (LOG.isDebugEnabled()) {
2329              LOG.debug("Got access token error in response to OP_BLOCK_CHECKSUM "
2330                  + "for file " + src + " for block " + block
2331                  + " from datanode " + datanodes[j]
2332                  + ". Will retry the block once.");
2333            }
2334            lastRetriedIndex = i;
2335            done = true; // actually it's not done; but we'll retry
2336            i--; // repeat at i-th block
2337            refetchBlocks = true;
2338            break;
2339          }
2340        } catch (IOException ie) {
2341          LOG.warn("src=" + src + ", datanodes["+j+"]=" + datanodes[j], ie);
2342        } finally {
2343          IOUtils.closeStream(in);
2344          IOUtils.closeStream(out);
2345        }
2346      }
2347
2348      if (!done) {
2349        throw new IOException("Fail to get block MD5 for " + block);
2350      }
2351    }
2352
2353    //compute file MD5
2354    final MD5Hash fileMD5 = MD5Hash.digest(md5out.getData()); 
2355    switch (crcType) {
2356      case CRC32:
2357        return new MD5MD5CRC32GzipFileChecksum(bytesPerCRC,
2358            crcPerBlock, fileMD5);
2359      case CRC32C:
2360        return new MD5MD5CRC32CastagnoliFileChecksum(bytesPerCRC,
2361            crcPerBlock, fileMD5);
2362      default:
2363        // If there is no block allocated for the file,
2364        // return one with the magic entry that matches what previous
2365        // hdfs versions return.
2366        if (locatedblocks.size() == 0) {
2367          return new MD5MD5CRC32GzipFileChecksum(0, 0, fileMD5);
2368        }
2369
2370        // we should never get here since the validity was checked
2371        // when getCrcType() was called above.
2372        return null;
2373    }
2374  }
2375
2376  /**
2377   * Connect to the given datanode's datantrasfer port, and return
2378   * the resulting IOStreamPair. This includes encryption wrapping, etc.
2379   */
2380  private IOStreamPair connectToDN(DatanodeInfo dn, int timeout,
2381      LocatedBlock lb) throws IOException {
2382    boolean success = false;
2383    Socket sock = null;
2384    try {
2385      sock = socketFactory.createSocket();
2386      String dnAddr = dn.getXferAddr(getConf().connectToDnViaHostname);
2387      if (LOG.isDebugEnabled()) {
2388        LOG.debug("Connecting to datanode " + dnAddr);
2389      }
2390      NetUtils.connect(sock, NetUtils.createSocketAddr(dnAddr), timeout);
2391      sock.setSoTimeout(timeout);
2392  
2393      OutputStream unbufOut = NetUtils.getOutputStream(sock);
2394      InputStream unbufIn = NetUtils.getInputStream(sock);
2395      IOStreamPair ret = saslClient.newSocketSend(sock, unbufOut, unbufIn, this,
2396        lb.getBlockToken(), dn);
2397      success = true;
2398      return ret;
2399    } finally {
2400      if (!success) {
2401        IOUtils.closeSocket(sock);
2402      }
2403    }
2404  }
2405  
2406  /**
2407   * Infer the checksum type for a replica by sending an OP_READ_BLOCK
2408   * for the first byte of that replica. This is used for compatibility
2409   * with older HDFS versions which did not include the checksum type in
2410   * OpBlockChecksumResponseProto.
2411   *
2412   * @param lb the located block
2413   * @param dn the connected datanode
2414   * @return the inferred checksum type
2415   * @throws IOException if an error occurs
2416   */
2417  private Type inferChecksumTypeByReading(LocatedBlock lb, DatanodeInfo dn)
2418      throws IOException {
2419    IOStreamPair pair = connectToDN(dn, dfsClientConf.socketTimeout, lb);
2420
2421    try {
2422      DataOutputStream out = new DataOutputStream(new BufferedOutputStream(pair.out,
2423          HdfsConstants.SMALL_BUFFER_SIZE));
2424      DataInputStream in = new DataInputStream(pair.in);
2425  
2426      new Sender(out).readBlock(lb.getBlock(), lb.getBlockToken(), clientName,
2427          0, 1, true, CachingStrategy.newDefaultStrategy());
2428      final BlockOpResponseProto reply =
2429          BlockOpResponseProto.parseFrom(PBHelper.vintPrefixed(in));
2430      String logInfo = "trying to read " + lb.getBlock() + " from datanode " + dn;
2431      DataTransferProtoUtil.checkBlockOpStatus(reply, logInfo);
2432
2433      return PBHelper.convert(reply.getReadOpChecksumInfo().getChecksum().getType());
2434    } finally {
2435      IOUtils.cleanup(null, pair.in, pair.out);
2436    }
2437  }
2438
2439  /**
2440   * Set permissions to a file or directory.
2441   * @param src path name.
2442   * @param permission permission to set to
2443   * 
2444   * @see ClientProtocol#setPermission(String, FsPermission)
2445   */
2446  public void setPermission(String src, FsPermission permission)
2447      throws IOException {
2448    checkOpen();
2449    TraceScope scope = getPathTraceScope("setPermission", src);
2450    try {
2451      namenode.setPermission(src, permission);
2452    } catch(RemoteException re) {
2453      throw re.unwrapRemoteException(AccessControlException.class,
2454                                     FileNotFoundException.class,
2455                                     SafeModeException.class,
2456                                     UnresolvedPathException.class,
2457                                     SnapshotAccessControlException.class);
2458    } finally {
2459      scope.close();
2460    }
2461  }
2462
2463  /**
2464   * Set file or directory owner.
2465   * @param src path name.
2466   * @param username user id.
2467   * @param groupname user group.
2468   * 
2469   * @see ClientProtocol#setOwner(String, String, String)
2470   */
2471  public void setOwner(String src, String username, String groupname)
2472      throws IOException {
2473    checkOpen();
2474    TraceScope scope = getPathTraceScope("setOwner", src);
2475    try {
2476      namenode.setOwner(src, username, groupname);
2477    } catch(RemoteException re) {
2478      throw re.unwrapRemoteException(AccessControlException.class,
2479                                     FileNotFoundException.class,
2480                                     SafeModeException.class,
2481                                     UnresolvedPathException.class,
2482                                     SnapshotAccessControlException.class);                                   
2483    } finally {
2484      scope.close();
2485    }
2486  }
2487
2488  private long[] callGetStats() throws IOException {
2489    checkOpen();
2490    TraceScope scope = Trace.startSpan("getStats", traceSampler);
2491    try {
2492      return namenode.getStats();
2493    } finally {
2494      scope.close();
2495    }
2496  }
2497
2498  /**
2499   * @see ClientProtocol#getStats()
2500   */
2501  public FsStatus getDiskStatus() throws IOException {
2502    long rawNums[] = callGetStats();
2503    return new FsStatus(rawNums[0], rawNums[1], rawNums[2]);
2504  }
2505
2506  /**
2507   * Returns count of blocks with no good replicas left. Normally should be 
2508   * zero.
2509   * @throws IOException
2510   */ 
2511  public long getMissingBlocksCount() throws IOException {
2512    return callGetStats()[ClientProtocol.GET_STATS_MISSING_BLOCKS_IDX];
2513  }
2514  
2515  /**
2516   * Returns count of blocks with replication factor 1 and have
2517   * lost the only replica.
2518   * @throws IOException
2519   */
2520  public long getMissingReplOneBlocksCount() throws IOException {
2521    return callGetStats()[ClientProtocol.
2522        GET_STATS_MISSING_REPL_ONE_BLOCKS_IDX];
2523  }
2524
2525  /**
2526   * Returns count of blocks with one of more replica missing.
2527   * @throws IOException
2528   */ 
2529  public long getUnderReplicatedBlocksCount() throws IOException {
2530    return callGetStats()[ClientProtocol.GET_STATS_UNDER_REPLICATED_IDX];
2531  }
2532  
2533  /**
2534   * Returns count of blocks with at least one replica marked corrupt. 
2535   * @throws IOException
2536   */ 
2537  public long getCorruptBlocksCount() throws IOException {
2538    return callGetStats()[ClientProtocol.GET_STATS_CORRUPT_BLOCKS_IDX];
2539  }
2540  
2541  /**
2542   * @return a list in which each entry describes a corrupt file/block
2543   * @throws IOException
2544   */
2545  public CorruptFileBlocks listCorruptFileBlocks(String path,
2546                                                 String cookie)
2547        throws IOException {
2548    checkOpen();
2549    TraceScope scope = getPathTraceScope("listCorruptFileBlocks", path);
2550    try {
2551      return namenode.listCorruptFileBlocks(path, cookie);
2552    } finally {
2553      scope.close();
2554    }
2555  }
2556
2557  public DatanodeInfo[] datanodeReport(DatanodeReportType type)
2558      throws IOException {
2559    checkOpen();
2560    TraceScope scope = Trace.startSpan("datanodeReport", traceSampler);
2561    try {
2562      return namenode.getDatanodeReport(type);
2563    } finally {
2564      scope.close();
2565    }
2566  }
2567    
2568  public DatanodeStorageReport[] getDatanodeStorageReport(
2569      DatanodeReportType type) throws IOException {
2570    checkOpen();
2571    TraceScope scope =
2572        Trace.startSpan("datanodeStorageReport", traceSampler);
2573    try {
2574      return namenode.getDatanodeStorageReport(type);
2575    } finally {
2576      scope.close();
2577    }
2578  }
2579
2580  /**
2581   * Enter, leave or get safe mode.
2582   * 
2583   * @see ClientProtocol#setSafeMode(HdfsConstants.SafeModeAction,boolean)
2584   */
2585  public boolean setSafeMode(SafeModeAction action) throws IOException {
2586    return setSafeMode(action, false);
2587  }
2588  
2589  /**
2590   * Enter, leave or get safe mode.
2591   * 
2592   * @param action
2593   *          One of SafeModeAction.GET, SafeModeAction.ENTER and
2594   *          SafeModeActiob.LEAVE
2595   * @param isChecked
2596   *          If true, then check only active namenode's safemode status, else
2597   *          check first namenode's status.
2598   * @see ClientProtocol#setSafeMode(HdfsConstants.SafeModeAction, boolean)
2599   */
2600  public boolean setSafeMode(SafeModeAction action, boolean isChecked) throws IOException{
2601    TraceScope scope =
2602        Trace.startSpan("setSafeMode", traceSampler);
2603    try {
2604      return namenode.setSafeMode(action, isChecked);
2605    } finally {
2606      scope.close();
2607    }
2608  }
2609 
2610  /**
2611   * Create one snapshot.
2612   * 
2613   * @param snapshotRoot The directory where the snapshot is to be taken
2614   * @param snapshotName Name of the snapshot
2615   * @return the snapshot path.
2616   * @see ClientProtocol#createSnapshot(String, String)
2617   */
2618  public String createSnapshot(String snapshotRoot, String snapshotName)
2619      throws IOException {
2620    checkOpen();
2621    TraceScope scope = Trace.startSpan("createSnapshot", traceSampler);
2622    try {
2623      return namenode.createSnapshot(snapshotRoot, snapshotName);
2624    } catch(RemoteException re) {
2625      throw re.unwrapRemoteException();
2626    } finally {
2627      scope.close();
2628    }
2629  }
2630  
2631  /**
2632   * Delete a snapshot of a snapshottable directory.
2633   * 
2634   * @param snapshotRoot The snapshottable directory that the 
2635   *                    to-be-deleted snapshot belongs to
2636   * @param snapshotName The name of the to-be-deleted snapshot
2637   * @throws IOException
2638   * @see ClientProtocol#deleteSnapshot(String, String)
2639   */
2640  public void deleteSnapshot(String snapshotRoot, String snapshotName)
2641      throws IOException {
2642    checkOpen();
2643    TraceScope scope = Trace.startSpan("deleteSnapshot", traceSampler);
2644    try {
2645      namenode.deleteSnapshot(snapshotRoot, snapshotName);
2646    } catch(RemoteException re) {
2647      throw re.unwrapRemoteException();
2648    } finally {
2649      scope.close();
2650    }
2651  }
2652  
2653  /**
2654   * Rename a snapshot.
2655   * @param snapshotDir The directory path where the snapshot was taken
2656   * @param snapshotOldName Old name of the snapshot
2657   * @param snapshotNewName New name of the snapshot
2658   * @throws IOException
2659   * @see ClientProtocol#renameSnapshot(String, String, String)
2660   */
2661  public void renameSnapshot(String snapshotDir, String snapshotOldName,
2662      String snapshotNewName) throws IOException {
2663    checkOpen();
2664    TraceScope scope = Trace.startSpan("renameSnapshot", traceSampler);
2665    try {
2666      namenode.renameSnapshot(snapshotDir, snapshotOldName, snapshotNewName);
2667    } catch(RemoteException re) {
2668      throw re.unwrapRemoteException();
2669    } finally {
2670      scope.close();
2671    }
2672  }
2673  
2674  /**
2675   * Get all the current snapshottable directories.
2676   * @return All the current snapshottable directories
2677   * @throws IOException
2678   * @see ClientProtocol#getSnapshottableDirListing()
2679   */
2680  public SnapshottableDirectoryStatus[] getSnapshottableDirListing()
2681      throws IOException {
2682    checkOpen();
2683    TraceScope scope = Trace.startSpan("getSnapshottableDirListing",
2684        traceSampler);
2685    try {
2686      return namenode.getSnapshottableDirListing();
2687    } catch(RemoteException re) {
2688      throw re.unwrapRemoteException();
2689    } finally {
2690      scope.close();
2691    }
2692  }
2693
2694  /**
2695   * Allow snapshot on a directory.
2696   * 
2697   * @see ClientProtocol#allowSnapshot(String snapshotRoot)
2698   */
2699  public void allowSnapshot(String snapshotRoot) throws IOException {
2700    checkOpen();
2701    TraceScope scope = Trace.startSpan("allowSnapshot", traceSampler);
2702    try {
2703      namenode.allowSnapshot(snapshotRoot);
2704    } catch (RemoteException re) {
2705      throw re.unwrapRemoteException();
2706    } finally {
2707      scope.close();
2708    }
2709  }
2710  
2711  /**
2712   * Disallow snapshot on a directory.
2713   * 
2714   * @see ClientProtocol#disallowSnapshot(String snapshotRoot)
2715   */
2716  public void disallowSnapshot(String snapshotRoot) throws IOException {
2717    checkOpen();
2718    TraceScope scope = Trace.startSpan("disallowSnapshot", traceSampler);
2719    try {
2720      namenode.disallowSnapshot(snapshotRoot);
2721    } catch (RemoteException re) {
2722      throw re.unwrapRemoteException();
2723    } finally {
2724      scope.close();
2725    }
2726  }
2727  
2728  /**
2729   * Get the difference between two snapshots, or between a snapshot and the
2730   * current tree of a directory.
2731   * @see ClientProtocol#getSnapshotDiffReport(String, String, String)
2732   */
2733  public SnapshotDiffReport getSnapshotDiffReport(String snapshotDir,
2734      String fromSnapshot, String toSnapshot) throws IOException {
2735    checkOpen();
2736    TraceScope scope = Trace.startSpan("getSnapshotDiffReport", traceSampler);
2737    try {
2738      return namenode.getSnapshotDiffReport(snapshotDir,
2739          fromSnapshot, toSnapshot);
2740    } catch(RemoteException re) {
2741      throw re.unwrapRemoteException();
2742    } finally {
2743      scope.close();
2744    }
2745  }
2746
2747  public long addCacheDirective(
2748      CacheDirectiveInfo info, EnumSet<CacheFlag> flags) throws IOException {
2749    checkOpen();
2750    TraceScope scope = Trace.startSpan("addCacheDirective", traceSampler);
2751    try {
2752      return namenode.addCacheDirective(info, flags);
2753    } catch (RemoteException re) {
2754      throw re.unwrapRemoteException();
2755    } finally {
2756      scope.close();
2757    }
2758  }
2759  
2760  public void modifyCacheDirective(
2761      CacheDirectiveInfo info, EnumSet<CacheFlag> flags) throws IOException {
2762    checkOpen();
2763    TraceScope scope = Trace.startSpan("modifyCacheDirective", traceSampler);
2764    try {
2765      namenode.modifyCacheDirective(info, flags);
2766    } catch (RemoteException re) {
2767      throw re.unwrapRemoteException();
2768    } finally {
2769      scope.close();
2770    }
2771  }
2772
2773  public void removeCacheDirective(long id)
2774      throws IOException {
2775    checkOpen();
2776    TraceScope scope = Trace.startSpan("removeCacheDirective", traceSampler);
2777    try {
2778      namenode.removeCacheDirective(id);
2779    } catch (RemoteException re) {
2780      throw re.unwrapRemoteException();
2781    } finally {
2782      scope.close();
2783    }
2784  }
2785  
2786  public RemoteIterator<CacheDirectiveEntry> listCacheDirectives(
2787      CacheDirectiveInfo filter) throws IOException {
2788    return new CacheDirectiveIterator(namenode, filter, traceSampler);
2789  }
2790
2791  public void addCachePool(CachePoolInfo info) throws IOException {
2792    checkOpen();
2793    TraceScope scope = Trace.startSpan("addCachePool", traceSampler);
2794    try {
2795      namenode.addCachePool(info);
2796    } catch (RemoteException re) {
2797      throw re.unwrapRemoteException();
2798    } finally {
2799      scope.close();
2800    }
2801  }
2802
2803  public void modifyCachePool(CachePoolInfo info) throws IOException {
2804    checkOpen();
2805    TraceScope scope = Trace.startSpan("modifyCachePool", traceSampler);
2806    try {
2807      namenode.modifyCachePool(info);
2808    } catch (RemoteException re) {
2809      throw re.unwrapRemoteException();
2810    } finally {
2811      scope.close();
2812    }
2813  }
2814
2815  public void removeCachePool(String poolName) throws IOException {
2816    checkOpen();
2817    TraceScope scope = Trace.startSpan("removeCachePool", traceSampler);
2818    try {
2819      namenode.removeCachePool(poolName);
2820    } catch (RemoteException re) {
2821      throw re.unwrapRemoteException();
2822    } finally {
2823      scope.close();
2824    }
2825  }
2826
2827  public RemoteIterator<CachePoolEntry> listCachePools() throws IOException {
2828    return new CachePoolIterator(namenode, traceSampler);
2829  }
2830
2831  /**
2832   * Save namespace image.
2833   * 
2834   * @see ClientProtocol#saveNamespace()
2835   */
2836  void saveNamespace() throws AccessControlException, IOException {
2837    TraceScope scope = Trace.startSpan("saveNamespace", traceSampler);
2838    try {
2839      namenode.saveNamespace();
2840    } catch(RemoteException re) {
2841      throw re.unwrapRemoteException(AccessControlException.class);
2842    } finally {
2843      scope.close();
2844    }
2845  }
2846
2847  /**
2848   * Rolls the edit log on the active NameNode.
2849   * @return the txid of the new log segment 
2850   *
2851   * @see ClientProtocol#rollEdits()
2852   */
2853  long rollEdits() throws AccessControlException, IOException {
2854    TraceScope scope = Trace.startSpan("rollEdits", traceSampler);
2855    try {
2856      return namenode.rollEdits();
2857    } catch(RemoteException re) {
2858      throw re.unwrapRemoteException(AccessControlException.class);
2859    } finally {
2860      scope.close();
2861    }
2862  }
2863
2864  @VisibleForTesting
2865  ExtendedBlock getPreviousBlock(long fileId) {
2866    return filesBeingWritten.get(fileId).getBlock();
2867  }
2868  
2869  /**
2870   * enable/disable restore failed storage.
2871   * 
2872   * @see ClientProtocol#restoreFailedStorage(String arg)
2873   */
2874  boolean restoreFailedStorage(String arg)
2875      throws AccessControlException, IOException{
2876    TraceScope scope = Trace.startSpan("restoreFailedStorage", traceSampler);
2877    try {
2878      return namenode.restoreFailedStorage(arg);
2879    } finally {
2880      scope.close();
2881    }
2882  }
2883
2884  /**
2885   * Refresh the hosts and exclude files.  (Rereads them.)
2886   * See {@link ClientProtocol#refreshNodes()} 
2887   * for more details.
2888   * 
2889   * @see ClientProtocol#refreshNodes()
2890   */
2891  public void refreshNodes() throws IOException {
2892    TraceScope scope = Trace.startSpan("refreshNodes", traceSampler);
2893    try {
2894      namenode.refreshNodes();
2895    } finally {
2896      scope.close();
2897    }
2898  }
2899
2900  /**
2901   * Dumps DFS data structures into specified file.
2902   * 
2903   * @see ClientProtocol#metaSave(String)
2904   */
2905  public void metaSave(String pathname) throws IOException {
2906    TraceScope scope = Trace.startSpan("metaSave", traceSampler);
2907    try {
2908      namenode.metaSave(pathname);
2909    } finally {
2910      scope.close();
2911    }
2912  }
2913
2914  /**
2915   * Requests the namenode to tell all datanodes to use a new, non-persistent
2916   * bandwidth value for dfs.balance.bandwidthPerSec.
2917   * See {@link ClientProtocol#setBalancerBandwidth(long)} 
2918   * for more details.
2919   * 
2920   * @see ClientProtocol#setBalancerBandwidth(long)
2921   */
2922  public void setBalancerBandwidth(long bandwidth) throws IOException {
2923    TraceScope scope = Trace.startSpan("setBalancerBandwidth", traceSampler);
2924    try {
2925      namenode.setBalancerBandwidth(bandwidth);
2926    } finally {
2927      scope.close();
2928    }
2929  }
2930    
2931  /**
2932   * @see ClientProtocol#finalizeUpgrade()
2933   */
2934  public void finalizeUpgrade() throws IOException {
2935    TraceScope scope = Trace.startSpan("finalizeUpgrade", traceSampler);
2936    try {
2937      namenode.finalizeUpgrade();
2938    } finally {
2939      scope.close();
2940    }
2941  }
2942
2943  RollingUpgradeInfo rollingUpgrade(RollingUpgradeAction action) throws IOException {
2944    TraceScope scope = Trace.startSpan("rollingUpgrade", traceSampler);
2945    try {
2946      return namenode.rollingUpgrade(action);
2947    } finally {
2948      scope.close();
2949    }
2950  }
2951
2952  /**
2953   */
2954  @Deprecated
2955  public boolean mkdirs(String src) throws IOException {
2956    return mkdirs(src, null, true);
2957  }
2958
2959  /**
2960   * Create a directory (or hierarchy of directories) with the given
2961   * name and permission.
2962   *
2963   * @param src The path of the directory being created
2964   * @param permission The permission of the directory being created.
2965   * If permission == null, use {@link FsPermission#getDefault()}.
2966   * @param createParent create missing parent directory if true
2967   * 
2968   * @return True if the operation success.
2969   * 
2970   * @see ClientProtocol#mkdirs(String, FsPermission, boolean)
2971   */
2972  public boolean mkdirs(String src, FsPermission permission,
2973      boolean createParent) throws IOException {
2974    if (permission == null) {
2975      permission = FsPermission.getDefault();
2976    }
2977    FsPermission masked = permission.applyUMask(dfsClientConf.uMask);
2978    return primitiveMkdir(src, masked, createParent);
2979  }
2980
2981  /**
2982   * Same {{@link #mkdirs(String, FsPermission, boolean)} except
2983   * that the permissions has already been masked against umask.
2984   */
2985  public boolean primitiveMkdir(String src, FsPermission absPermission)
2986    throws IOException {
2987    return primitiveMkdir(src, absPermission, true);
2988  }
2989
2990  /**
2991   * Same {{@link #mkdirs(String, FsPermission, boolean)} except
2992   * that the permissions has already been masked against umask.
2993   */
2994  public boolean primitiveMkdir(String src, FsPermission absPermission, 
2995    boolean createParent)
2996    throws IOException {
2997    checkOpen();
2998    if (absPermission == null) {
2999      absPermission = 
3000        FsPermission.getDefault().applyUMask(dfsClientConf.uMask);
3001    } 
3002
3003    if(LOG.isDebugEnabled()) {
3004      LOG.debug(src + ": masked=" + absPermission);
3005    }
3006    TraceScope scope = Trace.startSpan("mkdir", traceSampler);
3007    try {
3008      return namenode.mkdirs(src, absPermission, createParent);
3009    } catch(RemoteException re) {
3010      throw re.unwrapRemoteException(AccessControlException.class,
3011                                     InvalidPathException.class,
3012                                     FileAlreadyExistsException.class,
3013                                     FileNotFoundException.class,
3014                                     ParentNotDirectoryException.class,
3015                                     SafeModeException.class,
3016                                     NSQuotaExceededException.class,
3017                                     DSQuotaExceededException.class,
3018                                     UnresolvedPathException.class,
3019                                     SnapshotAccessControlException.class);
3020    } finally {
3021      scope.close();
3022    }
3023  }
3024  
3025  /**
3026   * Get {@link ContentSummary} rooted at the specified directory.
3027   * @param src The string representation of the path
3028   * 
3029   * @see ClientProtocol#getContentSummary(String)
3030   */
3031  ContentSummary getContentSummary(String src) throws IOException {
3032    TraceScope scope = getPathTraceScope("getContentSummary", src);
3033    try {
3034      return namenode.getContentSummary(src);
3035    } catch(RemoteException re) {
3036      throw re.unwrapRemoteException(AccessControlException.class,
3037                                     FileNotFoundException.class,
3038                                     UnresolvedPathException.class);
3039    } finally {
3040      scope.close();
3041    }
3042  }
3043
3044  /**
3045   * Sets or resets quotas for a directory.
3046   * @see ClientProtocol#setQuota(String, long, long, StorageType)
3047   */
3048  void setQuota(String src, long namespaceQuota, long storagespaceQuota)
3049      throws IOException {
3050    // sanity check
3051    if ((namespaceQuota <= 0 && namespaceQuota != HdfsConstants.QUOTA_DONT_SET &&
3052         namespaceQuota != HdfsConstants.QUOTA_RESET) ||
3053        (storagespaceQuota <= 0 && storagespaceQuota != HdfsConstants.QUOTA_DONT_SET &&
3054         storagespaceQuota != HdfsConstants.QUOTA_RESET)) {
3055      throw new IllegalArgumentException("Invalid values for quota : " +
3056                                         namespaceQuota + " and " +
3057                                         storagespaceQuota);
3058                                         
3059    }
3060    TraceScope scope = getPathTraceScope("setQuota", src);
3061    try {
3062      // Pass null as storage type for traditional namespace/storagespace quota.
3063      namenode.setQuota(src, namespaceQuota, storagespaceQuota, null);
3064    } catch(RemoteException re) {
3065      throw re.unwrapRemoteException(AccessControlException.class,
3066                                     FileNotFoundException.class,
3067                                     NSQuotaExceededException.class,
3068                                     DSQuotaExceededException.class,
3069                                     UnresolvedPathException.class,
3070                                     SnapshotAccessControlException.class);
3071    } finally {
3072      scope.close();
3073    }
3074  }
3075
3076  /**
3077   * Sets or resets quotas by storage type for a directory.
3078   * @see ClientProtocol#setQuota(String, long, long, StorageType)
3079   */
3080  void setQuotaByStorageType(String src, StorageType type, long quota)
3081      throws IOException {
3082    if (quota <= 0 && quota != HdfsConstants.QUOTA_DONT_SET &&
3083        quota != HdfsConstants.QUOTA_RESET) {
3084      throw new IllegalArgumentException("Invalid values for quota :" +
3085        quota);
3086    }
3087    if (type == null) {
3088      throw new IllegalArgumentException("Invalid storage type(null)");
3089    }
3090    if (!type.supportTypeQuota()) {
3091      throw new IllegalArgumentException("Don't support Quota for storage type : "
3092        + type.toString());
3093    }
3094    TraceScope scope = getPathTraceScope("setQuotaByStorageType", src);
3095    try {
3096      namenode.setQuota(src, HdfsConstants.QUOTA_DONT_SET, quota, type);
3097    } catch (RemoteException re) {
3098      throw re.unwrapRemoteException(AccessControlException.class,
3099        FileNotFoundException.class,
3100        QuotaByStorageTypeExceededException.class,
3101        UnresolvedPathException.class,
3102        SnapshotAccessControlException.class);
3103    } finally {
3104      scope.close();
3105    }
3106  }
3107  /**
3108   * set the modification and access time of a file
3109   * 
3110   * @see ClientProtocol#setTimes(String, long, long)
3111   */
3112  public void setTimes(String src, long mtime, long atime) throws IOException {
3113    checkOpen();
3114    TraceScope scope = getPathTraceScope("setTimes", src);
3115    try {
3116      namenode.setTimes(src, mtime, atime);
3117    } catch(RemoteException re) {
3118      throw re.unwrapRemoteException(AccessControlException.class,
3119                                     FileNotFoundException.class,
3120                                     UnresolvedPathException.class,
3121                                     SnapshotAccessControlException.class);
3122    } finally {
3123      scope.close();
3124    }
3125  }
3126
3127  /**
3128   * @deprecated use {@link HdfsDataInputStream} instead.
3129   */
3130  @Deprecated
3131  public static class DFSDataInputStream extends HdfsDataInputStream {
3132
3133    public DFSDataInputStream(DFSInputStream in) throws IOException {
3134      super(in);
3135    }
3136  }
3137
3138  void reportChecksumFailure(String file, ExtendedBlock blk, DatanodeInfo dn) {
3139    DatanodeInfo [] dnArr = { dn };
3140    LocatedBlock [] lblocks = { new LocatedBlock(blk, dnArr) };
3141    reportChecksumFailure(file, lblocks);
3142  }
3143    
3144  // just reports checksum failure and ignores any exception during the report.
3145  void reportChecksumFailure(String file, LocatedBlock lblocks[]) {
3146    try {
3147      reportBadBlocks(lblocks);
3148    } catch (IOException ie) {
3149      LOG.info("Found corruption while reading " + file
3150          + ". Error repairing corrupt blocks. Bad blocks remain.", ie);
3151    }
3152  }
3153
3154  @Override
3155  public String toString() {
3156    return getClass().getSimpleName() + "[clientName=" + clientName
3157        + ", ugi=" + ugi + "]"; 
3158  }
3159
3160  public CachingStrategy getDefaultReadCachingStrategy() {
3161    return defaultReadCachingStrategy;
3162  }
3163
3164  public CachingStrategy getDefaultWriteCachingStrategy() {
3165    return defaultWriteCachingStrategy;
3166  }
3167
3168  public ClientContext getClientContext() {
3169    return clientContext;
3170  }
3171
3172  public void modifyAclEntries(String src, List<AclEntry> aclSpec)
3173      throws IOException {
3174    checkOpen();
3175    TraceScope scope = getPathTraceScope("modifyAclEntries", src);
3176    try {
3177      namenode.modifyAclEntries(src, aclSpec);
3178    } catch(RemoteException re) {
3179      throw re.unwrapRemoteException(AccessControlException.class,
3180                                     AclException.class,
3181                                     FileNotFoundException.class,
3182                                     NSQuotaExceededException.class,
3183                                     SafeModeException.class,
3184                                     SnapshotAccessControlException.class,
3185                                     UnresolvedPathException.class);
3186    } finally {
3187      scope.close();
3188    }
3189  }
3190
3191  public void removeAclEntries(String src, List<AclEntry> aclSpec)
3192      throws IOException {
3193    checkOpen();
3194    TraceScope scope = Trace.startSpan("removeAclEntries", traceSampler);
3195    try {
3196      namenode.removeAclEntries(src, aclSpec);
3197    } catch(RemoteException re) {
3198      throw re.unwrapRemoteException(AccessControlException.class,
3199                                     AclException.class,
3200                                     FileNotFoundException.class,
3201                                     NSQuotaExceededException.class,
3202                                     SafeModeException.class,
3203                                     SnapshotAccessControlException.class,
3204                                     UnresolvedPathException.class);
3205    } finally {
3206      scope.close();
3207    }
3208  }
3209
3210  public void removeDefaultAcl(String src) throws IOException {
3211    checkOpen();
3212    TraceScope scope = Trace.startSpan("removeDefaultAcl", traceSampler);
3213    try {
3214      namenode.removeDefaultAcl(src);
3215    } catch(RemoteException re) {
3216      throw re.unwrapRemoteException(AccessControlException.class,
3217                                     AclException.class,
3218                                     FileNotFoundException.class,
3219                                     NSQuotaExceededException.class,
3220                                     SafeModeException.class,
3221                                     SnapshotAccessControlException.class,
3222                                     UnresolvedPathException.class);
3223    } finally {
3224      scope.close();
3225    }
3226  }
3227
3228  public void removeAcl(String src) throws IOException {
3229    checkOpen();
3230    TraceScope scope = Trace.startSpan("removeAcl", traceSampler);
3231    try {
3232      namenode.removeAcl(src);
3233    } catch(RemoteException re) {
3234      throw re.unwrapRemoteException(AccessControlException.class,
3235                                     AclException.class,
3236                                     FileNotFoundException.class,
3237                                     NSQuotaExceededException.class,
3238                                     SafeModeException.class,
3239                                     SnapshotAccessControlException.class,
3240                                     UnresolvedPathException.class);
3241    } finally {
3242      scope.close();
3243    }
3244  }
3245
3246  public void setAcl(String src, List<AclEntry> aclSpec) throws IOException {
3247    checkOpen();
3248    TraceScope scope = Trace.startSpan("setAcl", traceSampler);
3249    try {
3250      namenode.setAcl(src, aclSpec);
3251    } catch(RemoteException re) {
3252      throw re.unwrapRemoteException(AccessControlException.class,
3253                                     AclException.class,
3254                                     FileNotFoundException.class,
3255                                     NSQuotaExceededException.class,
3256                                     SafeModeException.class,
3257                                     SnapshotAccessControlException.class,
3258                                     UnresolvedPathException.class);
3259    } finally {
3260      scope.close();
3261    }
3262  }
3263
3264  public AclStatus getAclStatus(String src) throws IOException {
3265    checkOpen();
3266    TraceScope scope = getPathTraceScope("getAclStatus", src);
3267    try {
3268      return namenode.getAclStatus(src);
3269    } catch(RemoteException re) {
3270      throw re.unwrapRemoteException(AccessControlException.class,
3271                                     AclException.class,
3272                                     FileNotFoundException.class,
3273                                     UnresolvedPathException.class);
3274    } finally {
3275      scope.close();
3276    }
3277  }
3278  
3279  public void createEncryptionZone(String src, String keyName)
3280    throws IOException {
3281    checkOpen();
3282    TraceScope scope = getPathTraceScope("createEncryptionZone", src);
3283    try {
3284      namenode.createEncryptionZone(src, keyName);
3285    } catch (RemoteException re) {
3286      throw re.unwrapRemoteException(AccessControlException.class,
3287                                     SafeModeException.class,
3288                                     UnresolvedPathException.class);
3289    } finally {
3290      scope.close();
3291    }
3292  }
3293
3294  public EncryptionZone getEZForPath(String src)
3295          throws IOException {
3296    checkOpen();
3297    TraceScope scope = getPathTraceScope("getEZForPath", src);
3298    try {
3299      return namenode.getEZForPath(src);
3300    } catch (RemoteException re) {
3301      throw re.unwrapRemoteException(AccessControlException.class,
3302                                     UnresolvedPathException.class);
3303    } finally {
3304      scope.close();
3305    }
3306  }
3307
3308  public RemoteIterator<EncryptionZone> listEncryptionZones()
3309      throws IOException {
3310    checkOpen();
3311    return new EncryptionZoneIterator(namenode, traceSampler);
3312  }
3313
3314  public void setXAttr(String src, String name, byte[] value, 
3315      EnumSet<XAttrSetFlag> flag) throws IOException {
3316    checkOpen();
3317    TraceScope scope = getPathTraceScope("setXAttr", src);
3318    try {
3319      namenode.setXAttr(src, XAttrHelper.buildXAttr(name, value), flag);
3320    } catch (RemoteException re) {
3321      throw re.unwrapRemoteException(AccessControlException.class,
3322                                     FileNotFoundException.class,
3323                                     NSQuotaExceededException.class,
3324                                     SafeModeException.class,
3325                                     SnapshotAccessControlException.class,
3326                                     UnresolvedPathException.class);
3327    } finally {
3328      scope.close();
3329    }
3330  }
3331  
3332  public byte[] getXAttr(String src, String name) throws IOException {
3333    checkOpen();
3334    TraceScope scope = getPathTraceScope("getXAttr", src);
3335    try {
3336      final List<XAttr> xAttrs = XAttrHelper.buildXAttrAsList(name);
3337      final List<XAttr> result = namenode.getXAttrs(src, xAttrs);
3338      return XAttrHelper.getFirstXAttrValue(result);
3339    } catch(RemoteException re) {
3340      throw re.unwrapRemoteException(AccessControlException.class,
3341                                     FileNotFoundException.class,
3342                                     UnresolvedPathException.class);
3343    } finally {
3344      scope.close();
3345    }
3346  }
3347  
3348  public Map<String, byte[]> getXAttrs(String src) throws IOException {
3349    checkOpen();
3350    TraceScope scope = getPathTraceScope("getXAttrs", src);
3351    try {
3352      return XAttrHelper.buildXAttrMap(namenode.getXAttrs(src, null));
3353    } catch(RemoteException re) {
3354      throw re.unwrapRemoteException(AccessControlException.class,
3355                                     FileNotFoundException.class,
3356                                     UnresolvedPathException.class);
3357    } finally {
3358      scope.close();
3359    }
3360  }
3361  
3362  public Map<String, byte[]> getXAttrs(String src, List<String> names) 
3363      throws IOException {
3364    checkOpen();
3365    TraceScope scope = getPathTraceScope("getXAttrs", src);
3366    try {
3367      return XAttrHelper.buildXAttrMap(namenode.getXAttrs(
3368          src, XAttrHelper.buildXAttrs(names)));
3369    } catch(RemoteException re) {
3370      throw re.unwrapRemoteException(AccessControlException.class,
3371                                     FileNotFoundException.class,
3372                                     UnresolvedPathException.class);
3373    } finally {
3374      scope.close();
3375    }
3376  }
3377  
3378  public List<String> listXAttrs(String src)
3379          throws IOException {
3380    checkOpen();
3381    TraceScope scope = getPathTraceScope("listXAttrs", src);
3382    try {
3383      final Map<String, byte[]> xattrs =
3384        XAttrHelper.buildXAttrMap(namenode.listXAttrs(src));
3385      return Lists.newArrayList(xattrs.keySet());
3386    } catch(RemoteException re) {
3387      throw re.unwrapRemoteException(AccessControlException.class,
3388                                     FileNotFoundException.class,
3389                                     UnresolvedPathException.class);
3390    } finally {
3391      scope.close();
3392    }
3393  }
3394
3395  public void removeXAttr(String src, String name) throws IOException {
3396    checkOpen();
3397    TraceScope scope = getPathTraceScope("removeXAttr", src);
3398    try {
3399      namenode.removeXAttr(src, XAttrHelper.buildXAttr(name));
3400    } catch(RemoteException re) {
3401      throw re.unwrapRemoteException(AccessControlException.class,
3402                                     FileNotFoundException.class,
3403                                     NSQuotaExceededException.class,
3404                                     SafeModeException.class,
3405                                     SnapshotAccessControlException.class,
3406                                     UnresolvedPathException.class);
3407    } finally {
3408      scope.close();
3409    }
3410  }
3411
3412  public void checkAccess(String src, FsAction mode) throws IOException {
3413    checkOpen();
3414    TraceScope scope = getPathTraceScope("checkAccess", src);
3415    try {
3416      namenode.checkAccess(src, mode);
3417    } catch (RemoteException re) {
3418      throw re.unwrapRemoteException(AccessControlException.class,
3419          FileNotFoundException.class,
3420          UnresolvedPathException.class);
3421    } finally {
3422      scope.close();
3423    }
3424  }
3425
3426  public DFSInotifyEventInputStream getInotifyEventStream() throws IOException {
3427    return new DFSInotifyEventInputStream(traceSampler, namenode);
3428  }
3429
3430  public DFSInotifyEventInputStream getInotifyEventStream(long lastReadTxid)
3431      throws IOException {
3432    return new DFSInotifyEventInputStream(traceSampler, namenode, lastReadTxid);
3433  }
3434
3435  @Override // RemotePeerFactory
3436  public Peer newConnectedPeer(InetSocketAddress addr,
3437      Token<BlockTokenIdentifier> blockToken, DatanodeID datanodeId)
3438      throws IOException {
3439    Peer peer = null;
3440    boolean success = false;
3441    Socket sock = null;
3442    try {
3443      sock = socketFactory.createSocket();
3444      NetUtils.connect(sock, addr,
3445        getRandomLocalInterfaceAddr(),
3446        dfsClientConf.socketTimeout);
3447      peer = TcpPeerServer.peerFromSocketAndKey(saslClient, sock, this,
3448          blockToken, datanodeId);
3449      peer.setReadTimeout(dfsClientConf.socketTimeout);
3450      success = true;
3451      return peer;
3452    } finally {
3453      if (!success) {
3454        IOUtils.cleanup(LOG, peer);
3455        IOUtils.closeSocket(sock);
3456      }
3457    }
3458  }
3459
3460  /**
3461   * Create hedged reads thread pool, HEDGED_READ_THREAD_POOL, if
3462   * it does not already exist.
3463   * @param num Number of threads for hedged reads thread pool.
3464   * If zero, skip hedged reads thread pool creation.
3465   */
3466  private synchronized void initThreadsNumForHedgedReads(int num) {
3467    if (num <= 0 || HEDGED_READ_THREAD_POOL != null) return;
3468    HEDGED_READ_THREAD_POOL = new ThreadPoolExecutor(1, num, 60,
3469        TimeUnit.SECONDS, new SynchronousQueue<Runnable>(),
3470        new Daemon.DaemonFactory() {
3471          private final AtomicInteger threadIndex =
3472            new AtomicInteger(0); 
3473          @Override
3474          public Thread newThread(Runnable r) {
3475            Thread t = super.newThread(r);
3476            t.setName("hedgedRead-" +
3477              threadIndex.getAndIncrement());
3478            return t;
3479          }
3480        },
3481        new ThreadPoolExecutor.CallerRunsPolicy() {
3482
3483      @Override
3484      public void rejectedExecution(Runnable runnable,
3485          ThreadPoolExecutor e) {
3486        LOG.info("Execution rejected, Executing in current thread");
3487        HEDGED_READ_METRIC.incHedgedReadOpsInCurThread();
3488        // will run in the current thread
3489        super.rejectedExecution(runnable, e);
3490      }
3491    });
3492    HEDGED_READ_THREAD_POOL.allowCoreThreadTimeOut(true);
3493    if (LOG.isDebugEnabled()) {
3494      LOG.debug("Using hedged reads; pool threads=" + num);
3495    }
3496  }
3497
3498  long getHedgedReadTimeout() {
3499    return this.hedgedReadThresholdMillis;
3500  }
3501
3502  @VisibleForTesting
3503  void setHedgedReadTimeout(long timeoutMillis) {
3504    this.hedgedReadThresholdMillis = timeoutMillis;
3505  }
3506
3507  ThreadPoolExecutor getHedgedReadsThreadPool() {
3508    return HEDGED_READ_THREAD_POOL;
3509  }
3510
3511  boolean isHedgedReadsEnabled() {
3512    return (HEDGED_READ_THREAD_POOL != null) &&
3513      HEDGED_READ_THREAD_POOL.getMaximumPoolSize() > 0;
3514  }
3515
3516  DFSHedgedReadMetrics getHedgedReadMetrics() {
3517    return HEDGED_READ_METRIC;
3518  }
3519
3520  public KeyProvider getKeyProvider() {
3521    return clientContext.getKeyProviderCache().get(conf);
3522  }
3523
3524  @VisibleForTesting
3525  public void setKeyProvider(KeyProvider provider) {
3526    try {
3527      clientContext.getKeyProviderCache().setKeyProvider(conf, provider);
3528    } catch (IOException e) {
3529     LOG.error("Could not set KeyProvider !!", e);
3530    }
3531  }
3532
3533  /**
3534   * Probe for encryption enabled on this filesystem.
3535   * See {@link DFSUtil#isHDFSEncryptionEnabled(Configuration)}
3536   * @return true if encryption is enabled
3537   */
3538  public boolean isHDFSEncryptionEnabled() {
3539    return DFSUtil.isHDFSEncryptionEnabled(this.conf);
3540  }
3541
3542  /**
3543   * Returns the SaslDataTransferClient configured for this DFSClient.
3544   *
3545   * @return SaslDataTransferClient configured for this DFSClient
3546   */
3547  public SaslDataTransferClient getSaslDataTransferClient() {
3548    return saslClient;
3549  }
3550
3551  private static final byte[] PATH = "path".getBytes(Charset.forName("UTF-8"));
3552
3553  TraceScope getPathTraceScope(String description, String path) {
3554    TraceScope scope = Trace.startSpan(description, traceSampler);
3555    Span span = scope.getSpan();
3556    if (span != null) {
3557      if (path != null) {
3558        span.addKVAnnotation(PATH,
3559            path.getBytes(Charset.forName("UTF-8")));
3560      }
3561    }
3562    return scope;
3563  }
3564
3565  private static final byte[] SRC = "src".getBytes(Charset.forName("UTF-8"));
3566
3567  private static final byte[] DST = "dst".getBytes(Charset.forName("UTF-8"));
3568
3569  TraceScope getSrcDstTraceScope(String description, String src, String dst) {
3570    TraceScope scope = Trace.startSpan(description, traceSampler);
3571    Span span = scope.getSpan();
3572    if (span != null) {
3573      if (src != null) {
3574        span.addKVAnnotation(SRC,
3575            src.getBytes(Charset.forName("UTF-8")));
3576      }
3577      if (dst != null) {
3578        span.addKVAnnotation(DST,
3579            dst.getBytes(Charset.forName("UTF-8")));
3580      }
3581    }
3582    return scope;
3583  }
3584}