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