|
@@ -63,11 +63,15 @@ import org.apache.hadoop.hdfs.server.datanode.DataNodeLayoutVersion;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.TestTransferRbw;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.fsdataset.FsDatasetSpi;
|
|
|
import org.apache.hadoop.hdfs.server.namenode.FSNamesystem;
|
|
|
+import org.apache.hadoop.hdfs.server.namenode.INodeFile;
|
|
|
import org.apache.hadoop.hdfs.server.namenode.NameNode;
|
|
|
import org.apache.hadoop.hdfs.server.namenode.ha
|
|
|
.ConfiguredFailoverProxyProvider;
|
|
|
import org.apache.hadoop.hdfs.server.protocol.DatanodeRegistration;
|
|
|
import org.apache.hadoop.hdfs.server.protocol.DatanodeStorage;
|
|
|
+import org.apache.hadoop.hdfs.server.protocol.ReceivedDeletedBlockInfo;
|
|
|
+import org.apache.hadoop.hdfs.server.protocol.ReceivedDeletedBlockInfo.BlockStatus;
|
|
|
+import org.apache.hadoop.hdfs.server.protocol.StorageReceivedDeletedBlocks;
|
|
|
import org.apache.hadoop.hdfs.tools.DFSAdmin;
|
|
|
import org.apache.hadoop.io.IOUtils;
|
|
|
import org.apache.hadoop.io.nativeio.NativeIO;
|
|
@@ -1636,4 +1640,67 @@ public class DFSTestUtil {
|
|
|
LayoutVersion.updateMap(DataNodeLayoutVersion.FEATURES,
|
|
|
new LayoutVersion.LayoutFeature[] { feature });
|
|
|
}
|
|
|
+
|
|
|
+ public static StorageReceivedDeletedBlocks[] makeReportForReceivedBlock(
|
|
|
+ Block block, BlockStatus blockStatus, DatanodeStorage storage) {
|
|
|
+ ReceivedDeletedBlockInfo[] receivedBlocks = new ReceivedDeletedBlockInfo[1];
|
|
|
+ receivedBlocks[0] = new ReceivedDeletedBlockInfo(block, blockStatus, null);
|
|
|
+ StorageReceivedDeletedBlocks[] reports = new StorageReceivedDeletedBlocks[1];
|
|
|
+ reports[0] = new StorageReceivedDeletedBlocks(storage, receivedBlocks);
|
|
|
+ return reports;
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Adds a block to a file.
|
|
|
+ * This method only manipulates NameNode
|
|
|
+ * states of the file and the block without injecting data to DataNode.
|
|
|
+ * It does mimic block reports.
|
|
|
+ * You should disable periodical heartbeat before use this.
|
|
|
+ * @param dataNodes List DataNodes to host the block
|
|
|
+ * @param previous Previous block in the file
|
|
|
+ * @param len block size
|
|
|
+ * @return The added block
|
|
|
+ */
|
|
|
+ public static Block addBlockToFile(
|
|
|
+ List<DataNode> dataNodes, DistributedFileSystem fs, FSNamesystem ns,
|
|
|
+ String file, INodeFile fileNode,
|
|
|
+ String clientName, ExtendedBlock previous, int len)
|
|
|
+ throws Exception {
|
|
|
+ fs.getClient().namenode.addBlock(file, clientName, previous, null,
|
|
|
+ fileNode.getId(), null);
|
|
|
+
|
|
|
+ final BlockInfo lastBlock =
|
|
|
+ fileNode.getLastBlock();
|
|
|
+ final int groupSize = fileNode.getBlockReplication();
|
|
|
+ assert dataNodes.size() >= groupSize;
|
|
|
+ // 1. RECEIVING_BLOCK IBR
|
|
|
+ for (int i = 0; i < groupSize; i++) {
|
|
|
+ DataNode dn = dataNodes.get(i);
|
|
|
+ final Block block = new Block(lastBlock.getBlockId() + i, 0,
|
|
|
+ lastBlock.getGenerationStamp());
|
|
|
+ DatanodeStorage storage = new DatanodeStorage(UUID.randomUUID().toString());
|
|
|
+ StorageReceivedDeletedBlocks[] reports = DFSTestUtil
|
|
|
+ .makeReportForReceivedBlock(block,
|
|
|
+ ReceivedDeletedBlockInfo.BlockStatus.RECEIVING_BLOCK, storage);
|
|
|
+ for (StorageReceivedDeletedBlocks report : reports) {
|
|
|
+ ns.processIncrementalBlockReport(dn.getDatanodeId(), report);
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ // 2. RECEIVED_BLOCK IBR
|
|
|
+ for (int i = 0; i < groupSize; i++) {
|
|
|
+ DataNode dn = dataNodes.get(i);
|
|
|
+ final Block block = new Block(lastBlock.getBlockId() + i,
|
|
|
+ len, lastBlock.getGenerationStamp());
|
|
|
+ DatanodeStorage storage = new DatanodeStorage(UUID.randomUUID().toString());
|
|
|
+ StorageReceivedDeletedBlocks[] reports = DFSTestUtil
|
|
|
+ .makeReportForReceivedBlock(block,
|
|
|
+ ReceivedDeletedBlockInfo.BlockStatus.RECEIVED_BLOCK, storage);
|
|
|
+ for (StorageReceivedDeletedBlocks report : reports) {
|
|
|
+ ns.processIncrementalBlockReport(dn.getDatanodeId(), report);
|
|
|
+ }
|
|
|
+ }
|
|
|
+ lastBlock.setNumBytes(len);
|
|
|
+ return lastBlock;
|
|
|
+ }
|
|
|
}
|