|
@@ -46,11 +46,11 @@ import java.nio.channels.ClosedChannelException;
|
|
|
import java.security.MessageDigest;
|
|
|
import java.util.Arrays;
|
|
|
|
|
|
+import com.google.common.annotations.VisibleForTesting;
|
|
|
import org.apache.commons.logging.Log;
|
|
|
import org.apache.hadoop.fs.StorageType;
|
|
|
import org.apache.hadoop.hdfs.DFSUtil;
|
|
|
import org.apache.hadoop.hdfs.ExtendedBlockId;
|
|
|
-import org.apache.hadoop.hdfs.HdfsConfiguration;
|
|
|
import org.apache.hadoop.hdfs.net.Peer;
|
|
|
import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
|
|
|
import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
|
|
@@ -620,7 +620,7 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
final long latestGenerationStamp,
|
|
|
DataChecksum requestedChecksum,
|
|
|
CachingStrategy cachingStrategy,
|
|
|
- final boolean allowLazyPersist,
|
|
|
+ boolean allowLazyPersist,
|
|
|
final boolean pinning,
|
|
|
final boolean[] targetPinnings) throws IOException {
|
|
|
previousOpClientName = clientname;
|
|
@@ -629,6 +629,8 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
final boolean isClient = !isDatanode;
|
|
|
final boolean isTransfer = stage == BlockConstructionStage.TRANSFER_RBW
|
|
|
|| stage == BlockConstructionStage.TRANSFER_FINALIZED;
|
|
|
+ allowLazyPersist = allowLazyPersist &&
|
|
|
+ (dnConf.getAllowNonLocalLazyPersist() || peer.isLocal());
|
|
|
long size = 0;
|
|
|
// check single target for transfer-RBW/Finalized
|
|
|
if (isTransfer && targets.length > 0) {
|
|
@@ -661,10 +663,7 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
+ localAddress);
|
|
|
|
|
|
// reply to upstream datanode or client
|
|
|
- final DataOutputStream replyOut = new DataOutputStream(
|
|
|
- new BufferedOutputStream(
|
|
|
- getOutputStream(),
|
|
|
- smallBufferSize));
|
|
|
+ final DataOutputStream replyOut = getBufferedOutputStream();
|
|
|
checkAccess(replyOut, isClient, block, blockToken,
|
|
|
Op.WRITE_BLOCK, BlockTokenIdentifier.AccessMode.WRITE);
|
|
|
|
|
@@ -679,7 +678,7 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
if (isDatanode ||
|
|
|
stage != BlockConstructionStage.PIPELINE_CLOSE_RECOVERY) {
|
|
|
// open a block receiver
|
|
|
- blockReceiver = new BlockReceiver(block, storageType, in,
|
|
|
+ blockReceiver = getBlockReceiver(block, storageType, in,
|
|
|
peer.getRemoteAddressString(),
|
|
|
peer.getLocalAddressString(),
|
|
|
stage, latestGenerationStamp, minBytesRcvd, maxBytesRcvd,
|
|
@@ -726,19 +725,18 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
smallBufferSize));
|
|
|
mirrorIn = new DataInputStream(unbufMirrorIn);
|
|
|
|
|
|
- // Do not propagate allowLazyPersist to downstream DataNodes.
|
|
|
if (targetPinnings != null && targetPinnings.length > 0) {
|
|
|
new Sender(mirrorOut).writeBlock(originalBlock, targetStorageTypes[0],
|
|
|
blockToken, clientname, targets, targetStorageTypes, srcDataNode,
|
|
|
stage, pipelineSize, minBytesRcvd, maxBytesRcvd,
|
|
|
latestGenerationStamp, requestedChecksum, cachingStrategy,
|
|
|
- false, targetPinnings[0], targetPinnings);
|
|
|
+ allowLazyPersist, targetPinnings[0], targetPinnings);
|
|
|
} else {
|
|
|
new Sender(mirrorOut).writeBlock(originalBlock, targetStorageTypes[0],
|
|
|
blockToken, clientname, targets, targetStorageTypes, srcDataNode,
|
|
|
stage, pipelineSize, minBytesRcvd, maxBytesRcvd,
|
|
|
latestGenerationStamp, requestedChecksum, cachingStrategy,
|
|
|
- false, false, targetPinnings);
|
|
|
+ allowLazyPersist, false, targetPinnings);
|
|
|
}
|
|
|
|
|
|
mirrorOut.flush();
|
|
@@ -853,8 +851,8 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
}
|
|
|
|
|
|
//update metrics
|
|
|
- datanode.metrics.addWriteBlockOp(elapsed());
|
|
|
- datanode.metrics.incrWritesFromClient(peer.isLocal(), size);
|
|
|
+ datanode.getMetrics().addWriteBlockOp(elapsed());
|
|
|
+ datanode.getMetrics().incrWritesFromClient(peer.isLocal(), size);
|
|
|
}
|
|
|
|
|
|
@Override
|
|
@@ -1161,7 +1159,7 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
DataChecksum remoteChecksum = DataTransferProtoUtil.fromProto(
|
|
|
checksumInfo.getChecksum());
|
|
|
// open a block receiver and check if the block does not exist
|
|
|
- blockReceiver = new BlockReceiver(block, storageType,
|
|
|
+ blockReceiver = getBlockReceiver(block, storageType,
|
|
|
proxyReply, proxySock.getRemoteSocketAddress().toString(),
|
|
|
proxySock.getLocalSocketAddress().toString(),
|
|
|
null, 0, 0, 0, "", null, datanode, remoteChecksum,
|
|
@@ -1216,6 +1214,39 @@ class DataXceiver extends Receiver implements Runnable {
|
|
|
datanode.metrics.addReplaceBlockOp(elapsed());
|
|
|
}
|
|
|
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Separated for testing.
|
|
|
+ */
|
|
|
+ @VisibleForTesting
|
|
|
+ BlockReceiver getBlockReceiver(
|
|
|
+ final ExtendedBlock block, final StorageType storageType,
|
|
|
+ final DataInputStream in,
|
|
|
+ final String inAddr, final String myAddr,
|
|
|
+ final BlockConstructionStage stage,
|
|
|
+ final long newGs, final long minBytesRcvd, final long maxBytesRcvd,
|
|
|
+ final String clientname, final DatanodeInfo srcDataNode,
|
|
|
+ final DataNode dn, DataChecksum requestedChecksum,
|
|
|
+ CachingStrategy cachingStrategy,
|
|
|
+ final boolean allowLazyPersist,
|
|
|
+ final boolean pinning) throws IOException {
|
|
|
+ return new BlockReceiver(block, storageType, in,
|
|
|
+ inAddr, myAddr, stage, newGs, minBytesRcvd, maxBytesRcvd,
|
|
|
+ clientname, srcDataNode, dn, requestedChecksum,
|
|
|
+ cachingStrategy, allowLazyPersist, pinning);
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Separated for testing.
|
|
|
+ * @return
|
|
|
+ */
|
|
|
+ @VisibleForTesting
|
|
|
+ DataOutputStream getBufferedOutputStream() {
|
|
|
+ return new DataOutputStream(
|
|
|
+ new BufferedOutputStream(getOutputStream(), smallBufferSize));
|
|
|
+ }
|
|
|
+
|
|
|
+
|
|
|
private long elapsed() {
|
|
|
return monotonicNow() - opStartTime;
|
|
|
}
|