|
@@ -20,7 +20,6 @@ package org.apache.hadoop.hdfs;
|
|
|
import static org.junit.Assert.assertFalse;
|
|
|
import static org.junit.Assert.assertTrue;
|
|
|
|
|
|
-import java.io.DataOutputStream;
|
|
|
import java.io.IOException;
|
|
|
import java.security.PrivilegedExceptionAction;
|
|
|
import java.util.HashMap;
|
|
@@ -31,80 +30,25 @@ import java.util.concurrent.Future;
|
|
|
import org.apache.commons.logging.Log;
|
|
|
import org.apache.commons.logging.LogFactory;
|
|
|
import org.apache.hadoop.conf.Configuration;
|
|
|
-import org.apache.hadoop.fs.FileStatus;
|
|
|
+import org.apache.hadoop.fs.CommonConfigurationKeys;
|
|
|
import org.apache.hadoop.fs.FileSystem;
|
|
|
import org.apache.hadoop.fs.Path;
|
|
|
import org.apache.hadoop.fs.Options.Rename;
|
|
|
import org.apache.hadoop.hdfs.protocol.LocatedBlocks;
|
|
|
import org.apache.hadoop.hdfs.server.blockmanagement.BlockManager;
|
|
|
import org.apache.hadoop.hdfs.server.namenode.NameNodeAdapter;
|
|
|
+import org.apache.hadoop.ipc.AsyncCallLimitExceededException;
|
|
|
import org.apache.hadoop.security.UserGroupInformation;
|
|
|
-import org.junit.After;
|
|
|
-import org.junit.Before;
|
|
|
import org.junit.Test;
|
|
|
|
|
|
public class TestAsyncDFSRename {
|
|
|
- final Path asyncRenameDir = new Path("/test/async_rename/");
|
|
|
public static final Log LOG = LogFactory.getLog(TestAsyncDFSRename.class);
|
|
|
- final private static Configuration CONF = new HdfsConfiguration();
|
|
|
-
|
|
|
- final private static String GROUP1_NAME = "group1";
|
|
|
- final private static String GROUP2_NAME = "group2";
|
|
|
- final private static String USER1_NAME = "user1";
|
|
|
- private static final UserGroupInformation USER1;
|
|
|
-
|
|
|
- private MiniDFSCluster gCluster;
|
|
|
-
|
|
|
- static {
|
|
|
- // explicitly turn on permission checking
|
|
|
- CONF.setBoolean(DFSConfigKeys.DFS_PERMISSIONS_ENABLED_KEY, true);
|
|
|
-
|
|
|
- // create fake mapping for the groups
|
|
|
- Map<String, String[]> u2g_map = new HashMap<String, String[]>(1);
|
|
|
- u2g_map.put(USER1_NAME, new String[] { GROUP1_NAME, GROUP2_NAME });
|
|
|
- DFSTestUtil.updateConfWithFakeGroupMapping(CONF, u2g_map);
|
|
|
-
|
|
|
- // Initiate all four users
|
|
|
- USER1 = UserGroupInformation.createUserForTesting(USER1_NAME, new String[] {
|
|
|
- GROUP1_NAME, GROUP2_NAME });
|
|
|
- }
|
|
|
-
|
|
|
- @Before
|
|
|
- public void setUp() throws IOException {
|
|
|
- gCluster = new MiniDFSCluster.Builder(CONF).numDataNodes(3).build();
|
|
|
- gCluster.waitActive();
|
|
|
- }
|
|
|
-
|
|
|
- @After
|
|
|
- public void tearDown() throws IOException {
|
|
|
- if (gCluster != null) {
|
|
|
- gCluster.shutdown();
|
|
|
- gCluster = null;
|
|
|
- }
|
|
|
- }
|
|
|
-
|
|
|
- static int countLease(MiniDFSCluster cluster) {
|
|
|
- return TestDFSRename.countLease(cluster);
|
|
|
- }
|
|
|
-
|
|
|
- void list(DistributedFileSystem dfs, String name) throws IOException {
|
|
|
- FileSystem.LOG.info("\n\n" + name);
|
|
|
- for (FileStatus s : dfs.listStatus(asyncRenameDir)) {
|
|
|
- FileSystem.LOG.info("" + s.getPath());
|
|
|
- }
|
|
|
- }
|
|
|
-
|
|
|
- static void createFile(DistributedFileSystem dfs, Path f) throws IOException {
|
|
|
- DataOutputStream a_out = dfs.create(f);
|
|
|
- a_out.writeBytes("something");
|
|
|
- a_out.close();
|
|
|
- }
|
|
|
|
|
|
/**
|
|
|
* Check the blocks of dst file are cleaned after rename with overwrite
|
|
|
* Restart NN to check the rename successfully
|
|
|
*/
|
|
|
- @Test
|
|
|
+ @Test(timeout = 60000)
|
|
|
public void testAsyncRenameWithOverwrite() throws Exception {
|
|
|
final short replFactor = 2;
|
|
|
final long blockSize = 512;
|
|
@@ -169,38 +113,134 @@ public class TestAsyncDFSRename {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- @Test
|
|
|
- public void testConcurrentAsyncRenameWithOverwrite() throws Exception {
|
|
|
+ @Test(timeout = 60000)
|
|
|
+ public void testCallGetReturnValueMultipleTimes() throws Exception {
|
|
|
final short replFactor = 2;
|
|
|
final long blockSize = 512;
|
|
|
final Path renameDir = new Path(
|
|
|
- "/test/concurrent_reanme_with_overwrite_dir/");
|
|
|
+ "/test/testCallGetReturnValueMultipleTimes/");
|
|
|
+ final Configuration conf = new HdfsConfiguration();
|
|
|
+ conf.setInt(CommonConfigurationKeys.IPC_CLIENT_ASYNC_CALLS_MAX_KEY, 200);
|
|
|
+ final MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf)
|
|
|
+ .numDataNodes(2).build();
|
|
|
+ cluster.waitActive();
|
|
|
+ final DistributedFileSystem dfs = cluster.getFileSystem();
|
|
|
+ final AsyncDistributedFileSystem adfs = dfs.getAsyncDistributedFileSystem();
|
|
|
+ final int count = 100;
|
|
|
+ long fileLen = blockSize * 3;
|
|
|
+ final Map<Integer, Future<Void>> returnFutures = new HashMap<Integer, Future<Void>>();
|
|
|
+
|
|
|
+ assertTrue(dfs.mkdirs(renameDir));
|
|
|
+
|
|
|
+ try {
|
|
|
+ // concurrently invoking many rename
|
|
|
+ for (int i = 0; i < count; i++) {
|
|
|
+ Path src = new Path(renameDir, "src" + i);
|
|
|
+ Path dst = new Path(renameDir, "dst" + i);
|
|
|
+ DFSTestUtil.createFile(dfs, src, fileLen, replFactor, 1);
|
|
|
+ DFSTestUtil.createFile(dfs, dst, fileLen, replFactor, 1);
|
|
|
+ Future<Void> returnFuture = adfs.rename(src, dst, Rename.OVERWRITE);
|
|
|
+ returnFutures.put(i, returnFuture);
|
|
|
+ }
|
|
|
+
|
|
|
+ for (int i = 0; i < 5; i++) {
|
|
|
+ verifyCallGetReturnValueMultipleTimes(returnFutures, count, cluster,
|
|
|
+ renameDir, dfs);
|
|
|
+ }
|
|
|
+ } finally {
|
|
|
+ if (dfs != null) {
|
|
|
+ dfs.close();
|
|
|
+ }
|
|
|
+ if (cluster != null) {
|
|
|
+ cluster.shutdown();
|
|
|
+ }
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ private void verifyCallGetReturnValueMultipleTimes(
|
|
|
+ Map<Integer, Future<Void>> returnFutures, int count,
|
|
|
+ MiniDFSCluster cluster, Path renameDir, DistributedFileSystem dfs)
|
|
|
+ throws InterruptedException, ExecutionException, IOException {
|
|
|
+ // wait for completing the calls
|
|
|
+ for (int i = 0; i < count; i++) {
|
|
|
+ returnFutures.get(i).get();
|
|
|
+ }
|
|
|
+
|
|
|
+ // Restart NN and check the rename successfully
|
|
|
+ cluster.restartNameNodes();
|
|
|
+
|
|
|
+ // very the src dir should not exist, dst should
|
|
|
+ for (int i = 0; i < count; i++) {
|
|
|
+ Path src = new Path(renameDir, "src" + i);
|
|
|
+ Path dst = new Path(renameDir, "dst" + i);
|
|
|
+ assertFalse(dfs.exists(src));
|
|
|
+ assertTrue(dfs.exists(dst));
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test(timeout = 120000)
|
|
|
+ public void testAggressiveConcurrentAsyncRenameWithOverwrite()
|
|
|
+ throws Exception {
|
|
|
+ internalTestConcurrentAsyncRenameWithOverwrite(100,
|
|
|
+ "testAggressiveConcurrentAsyncRenameWithOverwrite");
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test(timeout = 60000)
|
|
|
+ public void testConservativeConcurrentAsyncRenameWithOverwrite()
|
|
|
+ throws Exception {
|
|
|
+ internalTestConcurrentAsyncRenameWithOverwrite(10000,
|
|
|
+ "testConservativeConcurrentAsyncRenameWithOverwrite");
|
|
|
+ }
|
|
|
+
|
|
|
+ private void internalTestConcurrentAsyncRenameWithOverwrite(
|
|
|
+ final int asyncCallLimit, final String basePath) throws Exception {
|
|
|
+ final short replFactor = 2;
|
|
|
+ final long blockSize = 512;
|
|
|
+ final Path renameDir = new Path(String.format("/test/%s/", basePath));
|
|
|
Configuration conf = new HdfsConfiguration();
|
|
|
+ conf.setInt(CommonConfigurationKeys.IPC_CLIENT_ASYNC_CALLS_MAX_KEY,
|
|
|
+ asyncCallLimit);
|
|
|
MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2)
|
|
|
.build();
|
|
|
cluster.waitActive();
|
|
|
DistributedFileSystem dfs = cluster.getFileSystem();
|
|
|
AsyncDistributedFileSystem adfs = dfs.getAsyncDistributedFileSystem();
|
|
|
int count = 1000;
|
|
|
+ long fileLen = blockSize * 3;
|
|
|
+ int start = 0, end = 0;
|
|
|
+ Map<Integer, Future<Void>> returnFutures = new HashMap<Integer, Future<Void>>();
|
|
|
|
|
|
- try {
|
|
|
- long fileLen = blockSize * 3;
|
|
|
- assertTrue(dfs.mkdirs(renameDir));
|
|
|
-
|
|
|
- Map<Integer, Future<Void>> returnFutures = new HashMap<Integer, Future<Void>>();
|
|
|
+ assertTrue(dfs.mkdirs(renameDir));
|
|
|
|
|
|
+ try {
|
|
|
// concurrently invoking many rename
|
|
|
for (int i = 0; i < count; i++) {
|
|
|
Path src = new Path(renameDir, "src" + i);
|
|
|
Path dst = new Path(renameDir, "dst" + i);
|
|
|
DFSTestUtil.createFile(dfs, src, fileLen, replFactor, 1);
|
|
|
DFSTestUtil.createFile(dfs, dst, fileLen, replFactor, 1);
|
|
|
- Future<Void> returnFuture = adfs.rename(src, dst, Rename.OVERWRITE);
|
|
|
- returnFutures.put(i, returnFuture);
|
|
|
+ for (;;) {
|
|
|
+ try {
|
|
|
+ LOG.info("rename #" + i);
|
|
|
+ Future<Void> returnFuture = adfs.rename(src, dst, Rename.OVERWRITE);
|
|
|
+ returnFutures.put(i, returnFuture);
|
|
|
+ break;
|
|
|
+ } catch (AsyncCallLimitExceededException e) {
|
|
|
+ /**
|
|
|
+ * reached limit of async calls, fetch results of finished async
|
|
|
+ * calls to let follow-on calls go
|
|
|
+ */
|
|
|
+ LOG.error(e);
|
|
|
+ start = end;
|
|
|
+ end = i;
|
|
|
+ LOG.info(String.format("start=%d, end=%d, i=%d", start, end, i));
|
|
|
+ waitForReturnValues(returnFutures, start, end);
|
|
|
+ }
|
|
|
+ }
|
|
|
}
|
|
|
|
|
|
// wait for completing the calls
|
|
|
- for (int i = 0; i < count; i++) {
|
|
|
+ for (int i = start; i < count; i++) {
|
|
|
returnFutures.get(i).get();
|
|
|
}
|
|
|
|
|
@@ -215,26 +255,60 @@ public class TestAsyncDFSRename {
|
|
|
assertTrue(dfs.exists(dst));
|
|
|
}
|
|
|
} finally {
|
|
|
- dfs.delete(renameDir, true);
|
|
|
+ if (dfs != null) {
|
|
|
+ dfs.close();
|
|
|
+ }
|
|
|
if (cluster != null) {
|
|
|
cluster.shutdown();
|
|
|
}
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- @Test
|
|
|
+ private void waitForReturnValues(
|
|
|
+ final Map<Integer, Future<Void>> returnFutures, final int start,
|
|
|
+ final int end) throws InterruptedException, ExecutionException {
|
|
|
+ LOG.info(String.format("calling waitForReturnValues [%d, %d)", start, end));
|
|
|
+ for (int i = start; i < end; i++) {
|
|
|
+ LOG.info("calling Future#get #" + i);
|
|
|
+ returnFutures.get(i).get();
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ @Test(timeout = 60000)
|
|
|
public void testAsyncRenameWithException() throws Exception {
|
|
|
- FileSystem rootFs = FileSystem.get(CONF);
|
|
|
+ Configuration conf = new HdfsConfiguration();
|
|
|
+ String group1 = "group1";
|
|
|
+ String group2 = "group2";
|
|
|
+ String user1 = "user1";
|
|
|
+ UserGroupInformation ugi1;
|
|
|
+
|
|
|
+ // explicitly turn on permission checking
|
|
|
+ conf.setBoolean(DFSConfigKeys.DFS_PERMISSIONS_ENABLED_KEY, true);
|
|
|
+
|
|
|
+ // create fake mapping for the groups
|
|
|
+ Map<String, String[]> u2g_map = new HashMap<String, String[]>(1);
|
|
|
+ u2g_map.put(user1, new String[] { group1, group2 });
|
|
|
+ DFSTestUtil.updateConfWithFakeGroupMapping(conf, u2g_map);
|
|
|
+
|
|
|
+ // Initiate all four users
|
|
|
+ ugi1 = UserGroupInformation.createUserForTesting(user1, new String[] {
|
|
|
+ group1, group2 });
|
|
|
+
|
|
|
+ final MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf)
|
|
|
+ .numDataNodes(3).build();
|
|
|
+ cluster.waitActive();
|
|
|
+
|
|
|
+ FileSystem rootFs = FileSystem.get(conf);
|
|
|
final Path renameDir = new Path("/test/async_rename_exception/");
|
|
|
final Path src = new Path(renameDir, "src");
|
|
|
final Path dst = new Path(renameDir, "dst");
|
|
|
rootFs.mkdirs(src);
|
|
|
|
|
|
- AsyncDistributedFileSystem adfs = USER1
|
|
|
+ AsyncDistributedFileSystem adfs = ugi1
|
|
|
.doAs(new PrivilegedExceptionAction<AsyncDistributedFileSystem>() {
|
|
|
@Override
|
|
|
public AsyncDistributedFileSystem run() throws Exception {
|
|
|
- return gCluster.getFileSystem().getAsyncDistributedFileSystem();
|
|
|
+ return cluster.getFileSystem().getAsyncDistributedFileSystem();
|
|
|
}
|
|
|
});
|
|
|
|
|
@@ -242,16 +316,24 @@ public class TestAsyncDFSRename {
|
|
|
Future<Void> returnFuture = adfs.rename(src, dst, Rename.OVERWRITE);
|
|
|
returnFuture.get();
|
|
|
} catch (ExecutionException e) {
|
|
|
- checkPermissionDenied(e, src);
|
|
|
+ checkPermissionDenied(e, src, user1);
|
|
|
+ } finally {
|
|
|
+ if (rootFs != null) {
|
|
|
+ rootFs.close();
|
|
|
+ }
|
|
|
+ if (cluster != null) {
|
|
|
+ cluster.shutdown();
|
|
|
+ }
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- private void checkPermissionDenied(final Exception e, final Path dir) {
|
|
|
+ private void checkPermissionDenied(final Exception e, final Path dir,
|
|
|
+ final String user) {
|
|
|
assertTrue(e.getCause() instanceof ExecutionException);
|
|
|
assertTrue("Permission denied messages must carry AccessControlException",
|
|
|
e.getMessage().contains("AccessControlException"));
|
|
|
assertTrue("Permission denied messages must carry the username", e
|
|
|
- .getMessage().contains(USER1_NAME));
|
|
|
+ .getMessage().contains(user));
|
|
|
assertTrue("Permission denied messages must carry the path parent", e
|
|
|
.getMessage().contains(dir.getParent().toUri().getPath()));
|
|
|
}
|