|
@@ -23,11 +23,12 @@ import static org.junit.Assert.assertTrue;
|
|
|
import java.io.File;
|
|
|
import java.io.IOException;
|
|
|
import java.util.ArrayList;
|
|
|
+import java.util.Arrays;
|
|
|
import java.util.BitSet;
|
|
|
import java.util.HashMap;
|
|
|
import java.util.List;
|
|
|
import java.util.Map;
|
|
|
-import java.util.concurrent.ThreadLocalRandom;
|
|
|
+import java.util.Random;
|
|
|
|
|
|
import org.apache.commons.logging.Log;
|
|
|
import org.apache.commons.logging.LogFactory;
|
|
@@ -38,6 +39,7 @@ import org.apache.hadoop.hdfs.protocol.DatanodeInfo;
|
|
|
import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
|
|
|
import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
|
|
|
import org.apache.hadoop.hdfs.protocol.LocatedStripedBlock;
|
|
|
+import org.apache.hadoop.hdfs.server.blockmanagement.BlockManager;
|
|
|
import org.apache.hadoop.hdfs.server.blockmanagement.BlockManagerTestUtil;
|
|
|
import org.apache.hadoop.hdfs.server.blockmanagement.DatanodeStorageInfo;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.DataNode;
|
|
@@ -64,17 +66,25 @@ public class TestRecoverStripedFile {
|
|
|
|
|
|
static {
|
|
|
GenericTestUtils.setLogLevel(DFSClient.LOG, Level.ALL);
|
|
|
+ GenericTestUtils.setLogLevel(BlockManager.LOG, Level.ALL);
|
|
|
+ GenericTestUtils.setLogLevel(BlockManager.blockLog, Level.ALL);
|
|
|
+ }
|
|
|
+
|
|
|
+ enum RecoveryType {
|
|
|
+ DataOnly,
|
|
|
+ ParityOnly,
|
|
|
+ Any
|
|
|
}
|
|
|
|
|
|
private MiniDFSCluster cluster;
|
|
|
- private Configuration conf;
|
|
|
private DistributedFileSystem fs;
|
|
|
// Map: DatanodeID -> datanode index in cluster
|
|
|
- private Map<DatanodeID, Integer> dnMap = new HashMap<DatanodeID, Integer>();
|
|
|
+ private Map<DatanodeID, Integer> dnMap = new HashMap<>();
|
|
|
+ private final Random random = new Random();
|
|
|
|
|
|
@Before
|
|
|
public void setup() throws IOException {
|
|
|
- conf = new Configuration();
|
|
|
+ final Configuration conf = new Configuration();
|
|
|
conf.setLong(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, blockSize);
|
|
|
conf.setInt(DFSConfigKeys.DFS_DATANODE_STRIPED_READ_BUFFER_SIZE_KEY,
|
|
|
cellSize - 1);
|
|
@@ -104,75 +114,140 @@ public class TestRecoverStripedFile {
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverOneParityBlock() throws Exception {
|
|
|
int fileLen = 10 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverOneParityBlock", fileLen, 0, 1);
|
|
|
+ assertFileBlocksRecovery("/testRecoverOneParityBlock", fileLen,
|
|
|
+ RecoveryType.ParityOnly, 1);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverOneParityBlock1() throws Exception {
|
|
|
int fileLen = cellSize + cellSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverOneParityBlock1", fileLen, 0, 1);
|
|
|
+ assertFileBlocksRecovery("/testRecoverOneParityBlock1", fileLen,
|
|
|
+ RecoveryType.ParityOnly, 1);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverOneParityBlock2() throws Exception {
|
|
|
int fileLen = 1;
|
|
|
- assertFileBlocksRecovery("/testRecoverOneParityBlock2", fileLen, 0, 1);
|
|
|
+ assertFileBlocksRecovery("/testRecoverOneParityBlock2", fileLen,
|
|
|
+ RecoveryType.ParityOnly, 1);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverOneParityBlock3() throws Exception {
|
|
|
int fileLen = 3 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverOneParityBlock3", fileLen, 0, 1);
|
|
|
+ assertFileBlocksRecovery("/testRecoverOneParityBlock3", fileLen,
|
|
|
+ RecoveryType.ParityOnly, 1);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverThreeParityBlocks() throws Exception {
|
|
|
int fileLen = 10 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverThreeParityBlocks", fileLen, 0, 3);
|
|
|
+ assertFileBlocksRecovery("/testRecoverThreeParityBlocks", fileLen,
|
|
|
+ RecoveryType.ParityOnly, 3);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverThreeDataBlocks() throws Exception {
|
|
|
int fileLen = 10 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverThreeDataBlocks", fileLen, 1, 3);
|
|
|
+ assertFileBlocksRecovery("/testRecoverThreeDataBlocks", fileLen,
|
|
|
+ RecoveryType.DataOnly, 3);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverThreeDataBlocks1() throws Exception {
|
|
|
int fileLen = 3 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverThreeDataBlocks1", fileLen, 1, 3);
|
|
|
+ assertFileBlocksRecovery("/testRecoverThreeDataBlocks1", fileLen,
|
|
|
+ RecoveryType.DataOnly, 3);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverOneDataBlock() throws Exception {
|
|
|
int fileLen = 10 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverOneDataBlock", fileLen, 1, 1);
|
|
|
+ assertFileBlocksRecovery("/testRecoverOneDataBlock", fileLen,
|
|
|
+ RecoveryType.DataOnly, 1);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverOneDataBlock1() throws Exception {
|
|
|
int fileLen = cellSize + cellSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverOneDataBlock1", fileLen, 1, 1);
|
|
|
+ assertFileBlocksRecovery("/testRecoverOneDataBlock1", fileLen,
|
|
|
+ RecoveryType.DataOnly, 1);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverOneDataBlock2() throws Exception {
|
|
|
int fileLen = 1;
|
|
|
- assertFileBlocksRecovery("/testRecoverOneDataBlock2", fileLen, 1, 1);
|
|
|
+ assertFileBlocksRecovery("/testRecoverOneDataBlock2", fileLen,
|
|
|
+ RecoveryType.DataOnly, 1);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverAnyBlocks() throws Exception {
|
|
|
int fileLen = 3 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverAnyBlocks", fileLen, 2, 2);
|
|
|
+ assertFileBlocksRecovery("/testRecoverAnyBlocks", fileLen,
|
|
|
+ RecoveryType.Any, 2);
|
|
|
}
|
|
|
|
|
|
@Test(timeout = 120000)
|
|
|
public void testRecoverAnyBlocks1() throws Exception {
|
|
|
int fileLen = 10 * blockSize + blockSize/10;
|
|
|
- assertFileBlocksRecovery("/testRecoverAnyBlocks1", fileLen, 2, 3);
|
|
|
+ assertFileBlocksRecovery("/testRecoverAnyBlocks1", fileLen,
|
|
|
+ RecoveryType.Any, 3);
|
|
|
}
|
|
|
-
|
|
|
+
|
|
|
+ private int[] generateDeadDnIndices(RecoveryType type, int deadNum,
|
|
|
+ byte[] indices) {
|
|
|
+ List<Integer> deadList = new ArrayList<>(deadNum);
|
|
|
+ while (deadList.size() < deadNum) {
|
|
|
+ int dead = random.nextInt(indices.length);
|
|
|
+ boolean isOfType = true;
|
|
|
+ if (type == RecoveryType.DataOnly) {
|
|
|
+ isOfType = indices[dead] < dataBlkNum;
|
|
|
+ } else if (type == RecoveryType.ParityOnly) {
|
|
|
+ isOfType = indices[dead] >= dataBlkNum;
|
|
|
+ }
|
|
|
+ if (isOfType && !deadList.contains(dead)) {
|
|
|
+ deadList.add(dead);
|
|
|
+ }
|
|
|
+ }
|
|
|
+ int[] d = new int[deadNum];
|
|
|
+ for (int i = 0; i < deadNum; i++) {
|
|
|
+ d[i] = deadList.get(i);
|
|
|
+ }
|
|
|
+ return d;
|
|
|
+ }
|
|
|
+
|
|
|
+ private void shutdownDataNodes(DataNode dn) throws IOException {
|
|
|
+ /*
|
|
|
+ * Kill the datanode which contains one replica
|
|
|
+ * We need to make sure it dead in namenode: clear its update time and
|
|
|
+ * trigger NN to check heartbeat.
|
|
|
+ */
|
|
|
+ dn.shutdown();
|
|
|
+ cluster.setDataNodeDead(dn.getDatanodeId());
|
|
|
+ }
|
|
|
+
|
|
|
+ private int generateErrors(Map<ExtendedBlock, DataNode> corruptTargets,
|
|
|
+ RecoveryType type)
|
|
|
+ throws IOException {
|
|
|
+ int stoppedDN = 0;
|
|
|
+ for (Map.Entry<ExtendedBlock, DataNode> target : corruptTargets.entrySet()) {
|
|
|
+ if (stoppedDN == 0 || type != RecoveryType.DataOnly
|
|
|
+ || random.nextBoolean()) {
|
|
|
+ // stop at least one DN to trigger recovery
|
|
|
+ LOG.info("Note: stop DataNode " + target.getValue().getDisplayName()
|
|
|
+ + " with internal block " + target.getKey());
|
|
|
+ shutdownDataNodes(target.getValue());
|
|
|
+ stoppedDN++;
|
|
|
+ } else { // corrupt the data on the DN
|
|
|
+ LOG.info("Note: corrupt data on " + target.getValue().getDisplayName()
|
|
|
+ + " with internal block " + target.getKey());
|
|
|
+ cluster.corruptReplica(target.getValue(), target.getKey());
|
|
|
+ }
|
|
|
+ }
|
|
|
+ return stoppedDN;
|
|
|
+ }
|
|
|
+
|
|
|
/**
|
|
|
* Test the file blocks recovery.
|
|
|
* 1. Check the replica is recovered in the target datanode,
|
|
@@ -180,11 +255,7 @@ public class TestRecoverStripedFile {
|
|
|
* 2. Read the file and verify content.
|
|
|
*/
|
|
|
private void assertFileBlocksRecovery(String fileName, int fileLen,
|
|
|
- int recovery, int toRecoverBlockNum) throws Exception {
|
|
|
- if (recovery != 0 && recovery != 1 && recovery != 2) {
|
|
|
- Assert.fail("Invalid recovery: 0 is to recovery parity blocks,"
|
|
|
- + "1 is to recovery data blocks, 2 is any.");
|
|
|
- }
|
|
|
+ RecoveryType type, int toRecoverBlockNum) throws Exception {
|
|
|
if (toRecoverBlockNum < 1 || toRecoverBlockNum > parityBlkNum) {
|
|
|
Assert.fail("toRecoverBlockNum should be between 1 ~ " + parityBlkNum);
|
|
|
}
|
|
@@ -192,7 +263,7 @@ public class TestRecoverStripedFile {
|
|
|
Path file = new Path(fileName);
|
|
|
|
|
|
final byte[] data = new byte[fileLen];
|
|
|
- ThreadLocalRandom.current().nextBytes(data);
|
|
|
+ Arrays.fill(data, (byte) 1);
|
|
|
DFSTestUtil.writeFile(fs, file, data);
|
|
|
StripedFileTestUtil.waitBlockGroupsReported(fs, fileName);
|
|
|
|
|
@@ -209,26 +280,10 @@ public class TestRecoverStripedFile {
|
|
|
for (DatanodeInfo storageInfo : storageInfos) {
|
|
|
bitset.set(dnMap.get(storageInfo));
|
|
|
}
|
|
|
-
|
|
|
- int[] toDead = new int[toRecoverBlockNum];
|
|
|
- int n = 0;
|
|
|
- for (int i = 0; i < indices.length; i++) {
|
|
|
- if (n < toRecoverBlockNum) {
|
|
|
- if (recovery == 0) {
|
|
|
- if (indices[i] >= dataBlkNum) {
|
|
|
- toDead[n++] = i;
|
|
|
- }
|
|
|
- } else if (recovery == 1) {
|
|
|
- if (indices[i] < dataBlkNum) {
|
|
|
- toDead[n++] = i;
|
|
|
- }
|
|
|
- } else {
|
|
|
- toDead[n++] = i;
|
|
|
- }
|
|
|
- } else {
|
|
|
- break;
|
|
|
- }
|
|
|
- }
|
|
|
+
|
|
|
+ int[] dead = generateDeadDnIndices(type, toRecoverBlockNum, indices);
|
|
|
+ LOG.info("Note: indices == " + Arrays.toString(indices)
|
|
|
+ + ". Generate errors on datanodes: " + Arrays.toString(dead));
|
|
|
|
|
|
DatanodeInfo[] dataDNs = new DatanodeInfo[toRecoverBlockNum];
|
|
|
int[] deadDnIndices = new int[toRecoverBlockNum];
|
|
@@ -236,46 +291,41 @@ public class TestRecoverStripedFile {
|
|
|
File[] replicas = new File[toRecoverBlockNum];
|
|
|
File[] metadatas = new File[toRecoverBlockNum];
|
|
|
byte[][] replicaContents = new byte[toRecoverBlockNum][];
|
|
|
+ Map<ExtendedBlock, DataNode> errorMap = new HashMap<>(dead.length);
|
|
|
for (int i = 0; i < toRecoverBlockNum; i++) {
|
|
|
- dataDNs[i] = storageInfos[toDead[i]];
|
|
|
+ dataDNs[i] = storageInfos[dead[i]];
|
|
|
deadDnIndices[i] = dnMap.get(dataDNs[i]);
|
|
|
-
|
|
|
+
|
|
|
// Check the block replica file on deadDn before it dead.
|
|
|
blocks[i] = StripedBlockUtil.constructInternalBlock(
|
|
|
- lastBlock.getBlock(), cellSize, dataBlkNum, indices[toDead[i]]);
|
|
|
+ lastBlock.getBlock(), cellSize, dataBlkNum, indices[dead[i]]);
|
|
|
+ errorMap.put(blocks[i], cluster.getDataNodes().get(deadDnIndices[i]));
|
|
|
replicas[i] = cluster.getBlockFile(deadDnIndices[i], blocks[i]);
|
|
|
metadatas[i] = cluster.getBlockMetadataFile(deadDnIndices[i], blocks[i]);
|
|
|
// the block replica on the datanode should be the same as expected
|
|
|
assertEquals(replicas[i].length(),
|
|
|
StripedBlockUtil.getInternalBlockLength(
|
|
|
- lastBlock.getBlockSize(), cellSize, dataBlkNum, indices[toDead[i]]));
|
|
|
+ lastBlock.getBlockSize(), cellSize, dataBlkNum, indices[dead[i]]));
|
|
|
assertTrue(metadatas[i].getName().
|
|
|
endsWith(blocks[i].getGenerationStamp() + ".meta"));
|
|
|
+ LOG.info("replica " + i + " locates in file: " + replicas[i]);
|
|
|
replicaContents[i] = DFSTestUtil.readFileAsBytes(replicas[i]);
|
|
|
}
|
|
|
|
|
|
int cellsNum = (fileLen - 1) / cellSize + 1;
|
|
|
int groupSize = Math.min(cellsNum, dataBlkNum) + parityBlkNum;
|
|
|
|
|
|
- for (int i = 0; i < toRecoverBlockNum; i++) {
|
|
|
- /*
|
|
|
- * Kill the datanode which contains one replica
|
|
|
- * We need to make sure it dead in namenode: clear its update time and
|
|
|
- * trigger NN to check heartbeat.
|
|
|
- */
|
|
|
- DataNode dn = cluster.getDataNodes().get(deadDnIndices[i]);
|
|
|
- dn.shutdown();
|
|
|
- cluster.setDataNodeDead(dn.getDatanodeId());
|
|
|
- }
|
|
|
+ // shutdown datanodes or generate corruption
|
|
|
+ int stoppedDN = generateErrors(errorMap, type);
|
|
|
|
|
|
// Check the locatedBlocks of the file again
|
|
|
locatedBlocks = getLocatedBlocks(file);
|
|
|
lastBlock = (LocatedStripedBlock)locatedBlocks.getLastLocatedBlock();
|
|
|
storageInfos = lastBlock.getLocations();
|
|
|
- assertEquals(storageInfos.length, groupSize - toRecoverBlockNum);
|
|
|
+ assertEquals(storageInfos.length, groupSize - stoppedDN);
|
|
|
|
|
|
int[] targetDNs = new int[dnNum - groupSize];
|
|
|
- n = 0;
|
|
|
+ int n = 0;
|
|
|
for (int i = 0; i < dnNum; i++) {
|
|
|
if (!bitset.get(i)) { // not contain replica of the block.
|
|
|
targetDNs[n++] = i;
|
|
@@ -289,9 +339,11 @@ public class TestRecoverStripedFile {
|
|
|
// Check the replica on the new target node.
|
|
|
for (int i = 0; i < toRecoverBlockNum; i++) {
|
|
|
File replicaAfterRecovery = cluster.getBlockFile(targetDNs[i], blocks[i]);
|
|
|
+ LOG.info("replica after recovery " + replicaAfterRecovery);
|
|
|
File metadataAfterRecovery =
|
|
|
cluster.getBlockMetadataFile(targetDNs[i], blocks[i]);
|
|
|
assertEquals(replicaAfterRecovery.length(), replicas[i].length());
|
|
|
+ LOG.info("replica before " + replicas[i]);
|
|
|
assertTrue(metadataAfterRecovery.getName().
|
|
|
endsWith(blocks[i].getGenerationStamp() + ".meta"));
|
|
|
byte[] replicaContentAfterRecovery =
|
|
@@ -366,7 +418,7 @@ public class TestRecoverStripedFile {
|
|
|
BlockECRecoveryInfo invalidECInfo = new BlockECRecoveryInfo(
|
|
|
new ExtendedBlock("bp-id", 123456), dataDNs, dnStorageInfo, liveIndices,
|
|
|
ErasureCodingPolicyManager.getSystemDefaultPolicy());
|
|
|
- List<BlockECRecoveryInfo> ecTasks = new ArrayList<BlockECRecoveryInfo>();
|
|
|
+ List<BlockECRecoveryInfo> ecTasks = new ArrayList<>();
|
|
|
ecTasks.add(invalidECInfo);
|
|
|
dataNode.getErasureCodingWorker().processErasureCodingTasks(ecTasks);
|
|
|
}
|