|
@@ -22,7 +22,9 @@ import java.net.URI;
|
|
|
import java.util.*;
|
|
|
import java.util.concurrent.atomic.AtomicInteger;
|
|
|
|
|
|
+import com.google.common.base.Supplier;
|
|
|
import com.google.common.collect.Maps;
|
|
|
+
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
import org.apache.hadoop.fs.FSDataOutputStream;
|
|
|
import org.apache.hadoop.fs.Path;
|
|
@@ -45,9 +47,12 @@ import org.apache.hadoop.test.GenericTestUtils;
|
|
|
import org.apache.hadoop.util.ToolRunner;
|
|
|
import org.junit.Assert;
|
|
|
import org.junit.Test;
|
|
|
+import org.slf4j.Logger;
|
|
|
+import org.slf4j.LoggerFactory;
|
|
|
|
|
|
public class TestMover {
|
|
|
|
|
|
+ private static final Logger LOG = LoggerFactory.getLogger(TestMover.class);
|
|
|
static final int DEFAULT_BLOCK_SIZE = 100;
|
|
|
|
|
|
static {
|
|
@@ -409,4 +414,73 @@ public class TestMover {
|
|
|
cluster.shutdown();
|
|
|
}
|
|
|
}
|
|
|
+
|
|
|
+ @Test(timeout = 300000)
|
|
|
+ public void testMoverWhenStoragePolicyUnset() throws Exception {
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
+ initConf(conf);
|
|
|
+ final MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf)
|
|
|
+ .numDataNodes(1)
|
|
|
+ .storageTypes(
|
|
|
+ new StorageType[][] {{StorageType.DISK, StorageType.ARCHIVE}})
|
|
|
+ .build();
|
|
|
+ try {
|
|
|
+ cluster.waitActive();
|
|
|
+ final DistributedFileSystem dfs = cluster.getFileSystem();
|
|
|
+ final String file = "/testMoverWhenStoragePolicyUnset";
|
|
|
+ // write to DISK
|
|
|
+ DFSTestUtil.createFile(dfs, new Path(file), 1L, (short) 1, 0L);
|
|
|
+
|
|
|
+ // move to ARCHIVE
|
|
|
+ dfs.setStoragePolicy(new Path(file), "COLD");
|
|
|
+ int rc = ToolRunner.run(conf, new Mover.Cli(),
|
|
|
+ new String[] {"-p", file.toString()});
|
|
|
+ Assert.assertEquals("Movement to ARCHIVE should be successful", 0, rc);
|
|
|
+
|
|
|
+ // Wait till namenode notified about the block location details
|
|
|
+ waitForLocatedBlockWithArchiveStorageType(dfs, file, 1);
|
|
|
+
|
|
|
+ // verify before unset policy
|
|
|
+ LocatedBlock lb = dfs.getClient().getLocatedBlocks(file, 0).get(0);
|
|
|
+ Assert.assertTrue(StorageType.ARCHIVE == (lb.getStorageTypes())[0]);
|
|
|
+
|
|
|
+ // unset storage policy
|
|
|
+ dfs.unsetStoragePolicy(new Path(file));
|
|
|
+ rc = ToolRunner.run(conf, new Mover.Cli(),
|
|
|
+ new String[] {"-p", file.toString()});
|
|
|
+ Assert.assertEquals("Movement to DISK should be successful", 0, rc);
|
|
|
+
|
|
|
+ lb = dfs.getClient().getLocatedBlocks(file, 0).get(0);
|
|
|
+ Assert.assertTrue(StorageType.DISK == (lb.getStorageTypes())[0]);
|
|
|
+ } finally {
|
|
|
+ cluster.shutdown();
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ private void waitForLocatedBlockWithArchiveStorageType(
|
|
|
+ final DistributedFileSystem dfs, final String file,
|
|
|
+ final int expectedArchiveCount) throws Exception {
|
|
|
+ GenericTestUtils.waitFor(new Supplier<Boolean>() {
|
|
|
+ @Override
|
|
|
+ public Boolean get() {
|
|
|
+ LocatedBlock lb = null;
|
|
|
+ try {
|
|
|
+ lb = dfs.getClient().getLocatedBlocks(file, 0).get(0);
|
|
|
+ } catch (IOException e) {
|
|
|
+ LOG.error("Exception while getting located blocks", e);
|
|
|
+ return false;
|
|
|
+ }
|
|
|
+ int archiveCount = 0;
|
|
|
+ for (StorageType storageType : lb.getStorageTypes()) {
|
|
|
+ if (StorageType.ARCHIVE == storageType) {
|
|
|
+ archiveCount++;
|
|
|
+ }
|
|
|
+ }
|
|
|
+ LOG.info("Archive replica count, expected={} and actual={}",
|
|
|
+ expectedArchiveCount, archiveCount);
|
|
|
+ return expectedArchiveCount == archiveCount;
|
|
|
+ }
|
|
|
+ }, 100, 3000);
|
|
|
+ }
|
|
|
+
|
|
|
}
|