|
@@ -18,10 +18,14 @@
|
|
|
package org.apache.hadoop.hdfs.server.datanode.fsdataset.impl;
|
|
|
|
|
|
import com.google.common.collect.Lists;
|
|
|
+
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.fs.FileSystemTestHelper;
|
|
|
import org.apache.hadoop.fs.StorageType;
|
|
|
import org.apache.hadoop.hdfs.DFSConfigKeys;
|
|
|
+import org.apache.hadoop.hdfs.HdfsConfiguration;
|
|
|
+import org.apache.hadoop.hdfs.MiniDFSCluster;
|
|
|
+import org.apache.hadoop.hdfs.protocol.Block;
|
|
|
import org.apache.hadoop.hdfs.protocol.ExtendedBlock;
|
|
|
import org.apache.hadoop.hdfs.server.common.HdfsServerConstants;
|
|
|
import org.apache.hadoop.hdfs.server.common.Storage;
|
|
@@ -29,8 +33,11 @@ import org.apache.hadoop.hdfs.server.common.StorageInfo;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.BlockScanner;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.DNConf;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.DataNode;
|
|
|
+import org.apache.hadoop.hdfs.server.datanode.DataNodeTestUtils;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.DataStorage;
|
|
|
+import org.apache.hadoop.hdfs.server.datanode.FinalizedReplica;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.ReplicaHandler;
|
|
|
+import org.apache.hadoop.hdfs.server.datanode.ReplicaInfo;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.StorageLocation;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.fsdataset.FsVolumeReference;
|
|
|
import org.apache.hadoop.hdfs.server.datanode.fsdataset.RoundRobinVolumeChoosingPolicy;
|
|
@@ -88,6 +95,8 @@ public class TestFsDatasetImpl {
|
|
|
private DataNode datanode;
|
|
|
private DataStorage storage;
|
|
|
private FsDatasetImpl dataset;
|
|
|
+
|
|
|
+ private final static String BLOCKPOOL = "BP-TEST";
|
|
|
|
|
|
private static Storage.StorageDirectory createStorageDirectory(File root) {
|
|
|
Storage.StorageDirectory sd = new Storage.StorageDirectory(root);
|
|
@@ -334,4 +343,54 @@ public class TestFsDatasetImpl {
|
|
|
|
|
|
FsDatasetTestUtil.assertFileLockReleased(badDir.toString());
|
|
|
}
|
|
|
+
|
|
|
+ @Test
|
|
|
+ public void testDeletingBlocks() throws IOException {
|
|
|
+ MiniDFSCluster cluster = new MiniDFSCluster.Builder(new HdfsConfiguration()).build();
|
|
|
+ try {
|
|
|
+ cluster.waitActive();
|
|
|
+ DataNode dn = cluster.getDataNodes().get(0);
|
|
|
+
|
|
|
+ FsDatasetImpl ds = (FsDatasetImpl) DataNodeTestUtils.getFSDataset(dn);
|
|
|
+ FsVolumeImpl vol = ds.getVolumes().get(0);
|
|
|
+
|
|
|
+ ExtendedBlock eb;
|
|
|
+ ReplicaInfo info;
|
|
|
+ List<Block> blockList = new ArrayList<Block>();
|
|
|
+ for (int i = 1; i <= 63; i++) {
|
|
|
+ eb = new ExtendedBlock(BLOCKPOOL, i, 1, 1000 + i);
|
|
|
+ info = new FinalizedReplica(
|
|
|
+ eb.getLocalBlock(), vol, vol.getCurrentDir().getParentFile());
|
|
|
+ ds.volumeMap.add(BLOCKPOOL, info);
|
|
|
+ info.getBlockFile().createNewFile();
|
|
|
+ info.getMetaFile().createNewFile();
|
|
|
+ blockList.add(info);
|
|
|
+ }
|
|
|
+ ds.invalidate(BLOCKPOOL, blockList.toArray(new Block[0]));
|
|
|
+ try {
|
|
|
+ Thread.sleep(1000);
|
|
|
+ } catch (InterruptedException e) {
|
|
|
+ // Nothing to do
|
|
|
+ }
|
|
|
+ assertTrue(ds.isDeletingBlock(BLOCKPOOL, blockList.get(0).getBlockId()));
|
|
|
+
|
|
|
+ blockList.clear();
|
|
|
+ eb = new ExtendedBlock(BLOCKPOOL, 64, 1, 1064);
|
|
|
+ info = new FinalizedReplica(
|
|
|
+ eb.getLocalBlock(), vol, vol.getCurrentDir().getParentFile());
|
|
|
+ ds.volumeMap.add(BLOCKPOOL, info);
|
|
|
+ info.getBlockFile().createNewFile();
|
|
|
+ info.getMetaFile().createNewFile();
|
|
|
+ blockList.add(info);
|
|
|
+ ds.invalidate(BLOCKPOOL, blockList.toArray(new Block[0]));
|
|
|
+ try {
|
|
|
+ Thread.sleep(1000);
|
|
|
+ } catch (InterruptedException e) {
|
|
|
+ // Nothing to do
|
|
|
+ }
|
|
|
+ assertFalse(ds.isDeletingBlock(BLOCKPOOL, blockList.get(0).getBlockId()));
|
|
|
+ } finally {
|
|
|
+ cluster.shutdown();
|
|
|
+ }
|
|
|
+ }
|
|
|
}
|