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