You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by li...@apache.org on 2016/10/06 05:57:47 UTC

[1/2] hadoop git commit: HDFS-10893. Refactor TestDFSShell by setting up MiniDFSCluser once for all commands test. Contributed by Mingliang Liu

Repository: hadoop
Updated Branches:
  refs/heads/branch-2 434403a2a -> 14bacd2b9


http://git-wip-us.apache.org/repos/asf/hadoop/blob/14bacd2b/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSShell.java
----------------------------------------------------------------------
diff --git a/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSShell.java b/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSShell.java
index 6068978..88f0c95 100644
--- a/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSShell.java
+++ b/hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/TestDFSShell.java
@@ -66,6 +66,10 @@ import org.apache.hadoop.test.PathUtils;
 import org.apache.hadoop.util.ReflectionUtils;
 import org.apache.hadoop.util.StringUtils;
 import org.apache.hadoop.util.ToolRunner;
+import org.junit.rules.Timeout;
+import org.junit.AfterClass;
+import org.junit.BeforeClass;
+import org.junit.Rule;
 
 import static org.apache.hadoop.fs.CommonConfigurationKeysPublic.FS_TRASH_INTERVAL_KEY;
 import static org.apache.hadoop.fs.permission.AclEntryScope.ACCESS;
@@ -95,6 +99,37 @@ public class TestDFSShell {
   private static final byte[] RAW_A1_VALUE = new byte[]{0x32, 0x32, 0x32};
   private static final byte[] TRUSTED_A1_VALUE = new byte[]{0x31, 0x31, 0x31};
   private static final byte[] USER_A1_VALUE = new byte[]{0x31, 0x32, 0x33};
+  private static final int BLOCK_SIZE = 1024;
+
+  private static MiniDFSCluster miniCluster;
+  private static DistributedFileSystem dfs;
+
+  @BeforeClass
+  public static void setup() throws IOException {
+    final Configuration conf = new Configuration();
+    conf.setBoolean(DFSConfigKeys.DFS_PERMISSIONS_ENABLED_KEY, true);
+    conf.setInt(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, BLOCK_SIZE);
+    // set up the shared miniCluster directory so individual tests can launch
+    // new clusters without conflict
+    conf.set(MiniDFSCluster.HDFS_MINIDFS_BASEDIR,
+        GenericTestUtils.getTestDir("TestDFSShell").getAbsolutePath());
+    conf.setBoolean(DFSConfigKeys.DFS_NAMENODE_XATTRS_ENABLED_KEY, true);
+    conf.setBoolean(DFSConfigKeys.DFS_NAMENODE_ACLS_ENABLED_KEY, true);
+
+    miniCluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
+    miniCluster.waitActive();
+    dfs = miniCluster.getFileSystem();
+  }
+
+  @AfterClass
+  public static void tearDown() {
+    if (miniCluster != null) {
+      miniCluster.shutdown(true, true);
+    }
+  }
+
+  @Rule
+  public Timeout globalTimeout= new Timeout(30 * 1000); // 30s
 
   static Path writeFile(FileSystem fs, Path f) throws IOException {
     DataOutputStream out = fs.create(f);
@@ -146,102 +181,74 @@ public class TestDFSShell {
 
   @Test (timeout = 30000)
   public void testZeroSizeFile() throws IOException {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    FileSystem fs = cluster.getFileSystem();
-    assertTrue("Not a HDFS: "+fs.getUri(),
-               fs instanceof DistributedFileSystem);
-    final DistributedFileSystem dfs = (DistributedFileSystem)fs;
-
-    try {
-      //create a zero size file
-      final File f1 = new File(TEST_ROOT_DIR, "f1");
-      assertTrue(!f1.exists());
-      assertTrue(f1.createNewFile());
-      assertTrue(f1.exists());
-      assertTrue(f1.isFile());
-      assertEquals(0L, f1.length());
-      
-      //copy to remote
-      final Path root = mkdir(dfs, new Path("/test/zeroSizeFile"));
-      final Path remotef = new Path(root, "dst");
-      show("copy local " + f1 + " to remote " + remotef);
-      dfs.copyFromLocalFile(false, false, new Path(f1.getPath()), remotef);
-      
-      //getBlockSize() should not throw exception
-      show("Block size = " + dfs.getFileStatus(remotef).getBlockSize());
-
-      //copy back
-      final File f2 = new File(TEST_ROOT_DIR, "f2");
-      assertTrue(!f2.exists());
-      dfs.copyToLocalFile(remotef, new Path(f2.getPath()));
-      assertTrue(f2.exists());
-      assertTrue(f2.isFile());
-      assertEquals(0L, f2.length());
-  
-      f1.delete();
-      f2.delete();
-    } finally {
-      try {dfs.close();} catch (Exception e) {}
-      cluster.shutdown();
-    }
+    //create a zero size file
+    final File f1 = new File(TEST_ROOT_DIR, "f1");
+    assertTrue(!f1.exists());
+    assertTrue(f1.createNewFile());
+    assertTrue(f1.exists());
+    assertTrue(f1.isFile());
+    assertEquals(0L, f1.length());
+
+    //copy to remote
+    final Path root = mkdir(dfs, new Path("/test/zeroSizeFile"));
+    final Path remotef = new Path(root, "dst");
+    show("copy local " + f1 + " to remote " + remotef);
+    dfs.copyFromLocalFile(false, false, new Path(f1.getPath()), remotef);
+
+    //getBlockSize() should not throw exception
+    show("Block size = " + dfs.getFileStatus(remotef).getBlockSize());
+
+    //copy back
+    final File f2 = new File(TEST_ROOT_DIR, "f2");
+    assertTrue(!f2.exists());
+    dfs.copyToLocalFile(remotef, new Path(f2.getPath()));
+    assertTrue(f2.exists());
+    assertTrue(f2.isFile());
+    assertEquals(0L, f2.length());
+
+    f1.delete();
+    f2.delete();
   }
   
   @Test (timeout = 30000)
   public void testRecursiveRm() throws IOException {
-	  Configuration conf = new HdfsConfiguration();
-	  MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-	  FileSystem fs = cluster.getFileSystem();
-	  assertTrue("Not a HDFS: " + fs.getUri(), 
-			  fs instanceof DistributedFileSystem);
-	  try {
-      fs.mkdirs(new Path(new Path("parent"), "child"));
-      try {
-        fs.delete(new Path("parent"), false);
-        assert(false); // should never reach here.
-      } catch(IOException e) {
-         //should have thrown an exception
-      }
-      try {
-        fs.delete(new Path("parent"), true);
-      } catch(IOException e) {
-        assert(false);
-      }
-    } finally {  
-      try { fs.close();}catch(IOException e){};
-      cluster.shutdown();
+    final Path parent = new Path("/testRecursiveRm", "parent");
+    final Path child = new Path(parent, "child");
+    dfs.mkdirs(child);
+    dfs.mkdirs(new Path(new Path("parent"), "child"));
+    try {
+      dfs.delete(parent, false);
+      fail("Should have failed because dir is not empty");
+    } catch(IOException e) {
+       //should have thrown an exception
     }
+    dfs.delete(parent, true);
+    assertFalse(dfs.exists(parent));
   }
     
   @Test (timeout = 30000)
   public void testDu() throws IOException {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    DistributedFileSystem fs = cluster.getFileSystem();
     PrintStream psBackup = System.out;
     ByteArrayOutputStream out = new ByteArrayOutputStream();
     PrintStream psOut = new PrintStream(out);
     System.setOut(psOut);
-    FsShell shell = new FsShell();
-    shell.setConf(conf);
-    
+    FsShell shell = new FsShell(dfs.getConf());
     try {
-      cluster.waitActive();
-      Path myPath = new Path("/test/dir");
-      assertTrue(fs.mkdirs(myPath));
-      assertTrue(fs.exists(myPath));
-      Path myFile = new Path("/test/dir/file");
-      writeFile(fs, myFile);
-      assertTrue(fs.exists(myFile));
-      Path myFile2 = new Path("/test/dir/file2");
-      writeFile(fs, myFile2);
-      assertTrue(fs.exists(myFile2));
-      Long myFileLength = fs.getFileStatus(myFile).getLen();
-      Long myFile2Length = fs.getFileStatus(myFile2).getLen();
-      
+      Path myPath = new Path("/testDu/dir");
+      assertTrue(dfs.mkdirs(myPath));
+      assertTrue(dfs.exists(myPath));
+      Path myFile = new Path(myPath, "file");
+      writeFile(dfs, myFile);
+      assertTrue(dfs.exists(myFile));
+      Path myFile2 = new Path(myPath, "file2");
+      writeFile(dfs, myFile2);
+      assertTrue(dfs.exists(myFile2));
+      Long myFileLength = dfs.getFileStatus(myFile).getLen();
+      Long myFile2Length = dfs.getFileStatus(myFile2).getLen();
+
       String[] args = new String[2];
       args[0] = "-du";
-      args[1] = "/test/dir";
+      args[1] = myPath.toString();
       int val = -1;
       try {
         val = shell.run(args);
@@ -259,10 +266,10 @@ public class TestDFSShell {
       // Check that -du -s reports the state of the snapshot
       String snapshotName = "ss1";
       Path snapshotPath = new Path(myPath, ".snapshot/" + snapshotName);
-      fs.allowSnapshot(myPath);
-      assertThat(fs.createSnapshot(myPath, snapshotName), is(snapshotPath));
-      assertThat(fs.delete(myFile, false), is(true));
-      assertThat(fs.exists(myFile), is(false));
+      dfs.allowSnapshot(myPath);
+      assertThat(dfs.createSnapshot(myPath, snapshotName), is(snapshotPath));
+      assertThat(dfs.delete(myFile, false), is(true));
+      assertThat(dfs.exists(myFile), is(false));
 
       args = new String[3];
       args[0] = "-du";
@@ -282,7 +289,6 @@ public class TestDFSShell {
       assertThat(returnString, containsString(combinedLength.toString()));
     } finally {
       System.setOut(psBackup);
-      cluster.shutdown();
     }
                                   
   }
@@ -290,20 +296,14 @@ public class TestDFSShell {
   @Test (timeout = 180000)
   public void testDuSnapshots() throws IOException {
     final int replication = 2;
-    final Configuration conf = new HdfsConfiguration();
-    final MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf)
-        .numDataNodes(replication).build();
-    final DistributedFileSystem dfs = cluster.getFileSystem();
     final PrintStream psBackup = System.out;
     final ByteArrayOutputStream out = new ByteArrayOutputStream();
     final PrintStream psOut = new PrintStream(out);
-    final FsShell shell = new FsShell();
-    shell.setConf(conf);
+    final FsShell shell = new FsShell(dfs.getConf());
 
     try {
       System.setOut(psOut);
-      cluster.waitActive();
-      final Path parent = new Path("/test");
+      final Path parent = new Path("/testDuSnapshots");
       final Path dir = new Path(parent, "dir");
       mkdir(dfs, dir);
       final Path file = new Path(dir, "file");
@@ -406,27 +406,19 @@ public class TestDFSShell {
       out.reset();
     } finally {
       System.setOut(psBackup);
-      cluster.shutdown();
     }
   }
 
   @Test (timeout = 180000)
   public void testCountSnapshots() throws IOException {
-    final int replication = 2;
-    final Configuration conf = new HdfsConfiguration();
-    final MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf)
-        .numDataNodes(replication).build();
-    final DistributedFileSystem dfs = cluster.getFileSystem();
     final PrintStream psBackup = System.out;
     final ByteArrayOutputStream out = new ByteArrayOutputStream();
     final PrintStream psOut = new PrintStream(out);
     System.setOut(psOut);
-    final FsShell shell = new FsShell();
-    shell.setConf(conf);
+    final FsShell shell = new FsShell(dfs.getConf());
 
     try {
-      cluster.waitActive();
-      final Path parent = new Path("/test");
+      final Path parent = new Path("/testCountSnapshots");
       final Path dir = new Path(parent, "dir");
       mkdir(dfs, dir);
       final Path file = new Path(dir, "file");
@@ -498,117 +490,100 @@ public class TestDFSShell {
       out.reset();
     } finally {
       System.setOut(psBackup);
-      cluster.shutdown();
     }
   }
 
   @Test (timeout = 30000)
   public void testPut() throws IOException {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    FileSystem fs = cluster.getFileSystem();
-    assertTrue("Not a HDFS: "+fs.getUri(),
-               fs instanceof DistributedFileSystem);
-    final DistributedFileSystem dfs = (DistributedFileSystem)fs;
+    // remove left over crc files:
+    new File(TEST_ROOT_DIR, ".f1.crc").delete();
+    new File(TEST_ROOT_DIR, ".f2.crc").delete();
+    final File f1 = createLocalFile(new File(TEST_ROOT_DIR, "f1"));
+    final File f2 = createLocalFile(new File(TEST_ROOT_DIR, "f2"));
 
-    try {
-      // remove left over crc files:
-      new File(TEST_ROOT_DIR, ".f1.crc").delete();
-      new File(TEST_ROOT_DIR, ".f2.crc").delete();    
-      final File f1 = createLocalFile(new File(TEST_ROOT_DIR, "f1"));
-      final File f2 = createLocalFile(new File(TEST_ROOT_DIR, "f2"));
-  
-      final Path root = mkdir(dfs, new Path("/test/put"));
-      final Path dst = new Path(root, "dst");
-  
-      show("begin");
-      
-      final Thread copy2ndFileThread = new Thread() {
-        @Override
-        public void run() {
-          try {
-            show("copy local " + f2 + " to remote " + dst);
-            dfs.copyFromLocalFile(false, false, new Path(f2.getPath()), dst);
-          } catch (IOException ioe) {
-            show("good " + StringUtils.stringifyException(ioe));
-            return;
-          }
-          //should not be here, must got IOException
-          assertTrue(false);
+    final Path root = mkdir(dfs, new Path("/testPut"));
+    final Path dst = new Path(root, "dst");
+
+    show("begin");
+
+    final Thread copy2ndFileThread = new Thread() {
+      @Override
+      public void run() {
+        try {
+          show("copy local " + f2 + " to remote " + dst);
+          dfs.copyFromLocalFile(false, false, new Path(f2.getPath()), dst);
+        } catch (IOException ioe) {
+          show("good " + StringUtils.stringifyException(ioe));
+          return;
         }
-      };
-      
-      //use SecurityManager to pause the copying of f1 and begin copying f2
-      SecurityManager sm = System.getSecurityManager();
-      System.out.println("SecurityManager = " + sm);
-      System.setSecurityManager(new SecurityManager() {
-        private boolean firstTime = true;
-  
-        @Override
-        public void checkPermission(Permission perm) {
-          if (firstTime) {
-            Thread t = Thread.currentThread();
-            if (!t.toString().contains("DataNode")) {
-              String s = "" + Arrays.asList(t.getStackTrace());
-              if (s.contains("FileUtil.copyContent")) {
-                //pause at FileUtil.copyContent
-  
-                firstTime = false;
-                copy2ndFileThread.start();
-                try {Thread.sleep(5000);} catch (InterruptedException e) {}
-              }
+        //should not be here, must got IOException
+        assertTrue(false);
+      }
+    };
+
+    //use SecurityManager to pause the copying of f1 and begin copying f2
+    SecurityManager sm = System.getSecurityManager();
+    System.out.println("SecurityManager = " + sm);
+    System.setSecurityManager(new SecurityManager() {
+      private boolean firstTime = true;
+
+      @Override
+      public void checkPermission(Permission perm) {
+        if (firstTime) {
+          Thread t = Thread.currentThread();
+          if (!t.toString().contains("DataNode")) {
+            String s = "" + Arrays.asList(t.getStackTrace());
+            if (s.contains("FileUtil.copyContent")) {
+              //pause at FileUtil.copyContent
+
+              firstTime = false;
+              copy2ndFileThread.start();
+              try {Thread.sleep(5000);} catch (InterruptedException e) {}
             }
           }
         }
-      });
-      show("copy local " + f1 + " to remote " + dst);
-      dfs.copyFromLocalFile(false, false, new Path(f1.getPath()), dst);
-      show("done");
-  
-      try {copy2ndFileThread.join();} catch (InterruptedException e) { }
-      System.setSecurityManager(sm);
-
-      // copy multiple files to destination directory
-      final Path destmultiple = mkdir(dfs, new Path("/test/putmultiple"));
-      Path[] srcs = new Path[2];
-      srcs[0] = new Path(f1.getPath());
-      srcs[1] = new Path(f2.getPath());
-      dfs.copyFromLocalFile(false, false, srcs, destmultiple);
-      srcs[0] = new Path(destmultiple,"f1"); 
-      srcs[1] = new Path(destmultiple,"f2"); 
-      assertTrue(dfs.exists(srcs[0]));
-      assertTrue(dfs.exists(srcs[1]));
-
-      // move multiple files to destination directory
-      final Path destmultiple2 = mkdir(dfs, new Path("/test/movemultiple"));
-      srcs[0] = new Path(f1.getPath());
-      srcs[1] = new Path(f2.getPath());
-      dfs.moveFromLocalFile(srcs, destmultiple2);
-      assertFalse(f1.exists());
-      assertFalse(f2.exists());
-      srcs[0] = new Path(destmultiple2, "f1");
-      srcs[1] = new Path(destmultiple2, "f2");
-      assertTrue(dfs.exists(srcs[0]));
-      assertTrue(dfs.exists(srcs[1]));
-
-      f1.delete();
-      f2.delete();
-    } finally {
-      try {dfs.close();} catch (Exception e) {}
-      cluster.shutdown();
-    }
+      }
+    });
+    show("copy local " + f1 + " to remote " + dst);
+    dfs.copyFromLocalFile(false, false, new Path(f1.getPath()), dst);
+    show("done");
+
+    try {copy2ndFileThread.join();} catch (InterruptedException e) { }
+    System.setSecurityManager(sm);
+
+    // copy multiple files to destination directory
+    final Path destmultiple = mkdir(dfs, new Path(root, "putmultiple"));
+    Path[] srcs = new Path[2];
+    srcs[0] = new Path(f1.getPath());
+    srcs[1] = new Path(f2.getPath());
+    dfs.copyFromLocalFile(false, false, srcs, destmultiple);
+    srcs[0] = new Path(destmultiple,"f1");
+    srcs[1] = new Path(destmultiple,"f2");
+    assertTrue(dfs.exists(srcs[0]));
+    assertTrue(dfs.exists(srcs[1]));
+
+    // move multiple files to destination directory
+    final Path destmultiple2 = mkdir(dfs, new Path(root, "movemultiple"));
+    srcs[0] = new Path(f1.getPath());
+    srcs[1] = new Path(f2.getPath());
+    dfs.moveFromLocalFile(srcs, destmultiple2);
+    assertFalse(f1.exists());
+    assertFalse(f2.exists());
+    srcs[0] = new Path(destmultiple2, "f1");
+    srcs[1] = new Path(destmultiple2, "f2");
+    assertTrue(dfs.exists(srcs[0]));
+    assertTrue(dfs.exists(srcs[1]));
+
+    f1.delete();
+    f2.delete();
   }
 
 
   /** check command error outputs and exit statuses. */
   @Test (timeout = 30000)
   public void testErrOutPut() throws Exception {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = null;
     PrintStream bak = null;
     try {
-      cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-      FileSystem srcFs = cluster.getFileSystem();
       Path root = new Path("/nonexistentfile");
       bak = System.err;
       ByteArrayOutputStream out = new ByteArrayOutputStream();
@@ -625,8 +600,7 @@ public class TestDFSShell {
       out.reset();
       argv[0] = "-rm";
       argv[1] = root.toString();
-      FsShell shell = new FsShell();
-      shell.setConf(conf);
+      FsShell shell = new FsShell(dfs.getConf());
       ret = ToolRunner.run(shell, argv);
       assertEquals(" -rm returned 1 ", 1, ret);
       returned = out.toString();
@@ -667,7 +641,7 @@ public class TestDFSShell {
       ret = ToolRunner.run(shell, argv);
       assertEquals(" -lsr should fail ", 1, ret);
       out.reset();
-      srcFs.mkdirs(new Path("/testdir"));
+      dfs.mkdirs(new Path("/testdir"));
       argv[0] = "-ls";
       argv[1] = "/testdir";
       ret = ToolRunner.run(shell, argv);
@@ -688,7 +662,7 @@ public class TestDFSShell {
       assertTrue(" -mkdir returned File exists", 
           (returned.lastIndexOf("File exists") != -1));
       Path testFile = new Path("/testfile");
-      OutputStream outtmp = srcFs.create(testFile);
+      OutputStream outtmp = dfs.create(testFile);
       outtmp.write(testFile.toString().getBytes());
       outtmp.close();
       out.reset();
@@ -703,7 +677,7 @@ public class TestDFSShell {
       argv = new String[3];
       argv[0] = "-mv";
       argv[1] = "/testfile";
-      argv[2] = "file";
+      argv[2] = "/no-such-dir/file";
       ret = ToolRunner.run(shell, argv);
       assertEquals("mv failed to rename", 1,  ret);
       out.reset();
@@ -726,7 +700,7 @@ public class TestDFSShell {
       out.reset();
       argv = new String[1];
       argv[0] = "-du";
-      srcFs.mkdirs(srcFs.getHomeDirectory());
+      dfs.mkdirs(dfs.getHomeDirectory());
       ret = ToolRunner.run(shell, argv);
       returned = out.toString();
       assertEquals(" no error ", 0, ret);
@@ -745,9 +719,6 @@ public class TestDFSShell {
       if (bak != null) {
         System.setErr(bak);
       }
-      if (cluster != null) {
-        cluster.shutdown();
-      }
     }
   }
 
@@ -789,7 +760,7 @@ public class TestDFSShell {
     Configuration dstConf = new HdfsConfiguration();
     MiniDFSCluster srcCluster =  null;
     MiniDFSCluster dstCluster = null;
-    File bak = new File(PathUtils.getTestDir(getClass()), "dfs_tmp_uri");
+    File bak = new File(PathUtils.getTestDir(getClass()), "testURIPaths");
     bak.mkdirs();
     try{
       srcCluster = new MiniDFSCluster.Builder(srcConf).numDataNodes(2).build();
@@ -846,7 +817,7 @@ public class TestDFSShell {
       Path path = new Path(file);
       Path parent = new Path("/tmp");
       Path root = new Path("/");
-      TestDFSShell.writeFile(dstFs, path);
+      writeFile(dstFs, path);
       runCmd(shell, "-chgrp", "-R", "herbivores", dstFs.getUri().toString() +"/*");
       confirmOwner(null, "herbivores", dstFs, parent, path);
       runCmd(shell, "-chown", "-R", ":reptiles", dstFs.getUri().toString() + "/");
@@ -879,35 +850,25 @@ public class TestDFSShell {
    */
   @Test (timeout = 30000)
   public void testTail() throws Exception {
-    final int blockSize = 1024;
-    final int fileLen = 5 * blockSize;
-    final Configuration conf = new Configuration();
-    conf.setInt(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, blockSize);
-
-    try (MiniDFSCluster cluster =
-             new MiniDFSCluster.Builder(conf).numDataNodes(3).build()) {
-      cluster.waitActive();
-      final DistributedFileSystem dfs = cluster.getFileSystem();
-
-      // create a text file with multiple KB bytes (and multiple blocks)
-      final Path testFile = new Path("testTail", "file1");
-      final String text = RandomStringUtils.randomAscii(fileLen);
-      try (OutputStream pout = dfs.create(testFile)) {
-        pout.write(text.getBytes());
-      }
-      final ByteArrayOutputStream out = new ByteArrayOutputStream();
-      System.setOut(new PrintStream(out));
-      final String[] argv = new String[]{"-tail", testFile.toString()};
-      final int ret = ToolRunner.run(new FsShell(conf), argv);
-
-      assertEquals(Arrays.toString(argv) + " returned " + ret, 0, ret);
-      assertEquals("-tail returned " + out.size() + " bytes data, expected 1KB",
-          1024, out.size());
-      // tailed out last 1KB of the file content
-      assertArrayEquals("Tail output doesn't match input",
-          text.substring(fileLen - 1024).getBytes(), out.toByteArray());
-      out.reset();
+    final int fileLen = 5 * BLOCK_SIZE;
+    // create a text file with multiple KB bytes (and multiple blocks)
+    final Path testFile = new Path("testTail", "file1");
+    final String text = RandomStringUtils.randomAscii(fileLen);
+    try (OutputStream pout = dfs.create(testFile)) {
+      pout.write(text.getBytes());
     }
+    final ByteArrayOutputStream out = new ByteArrayOutputStream();
+    System.setOut(new PrintStream(out));
+    final String[] argv = new String[]{"-tail", testFile.toString()};
+    final int ret = ToolRunner.run(new FsShell(dfs.getConf()), argv);
+
+    assertEquals(Arrays.toString(argv) + " returned " + ret, 0, ret);
+    assertEquals("-tail returned " + out.size() + " bytes data, expected 1KB",
+        1024, out.size());
+    // tailed out last 1KB of the file content
+    assertArrayEquals("Tail output doesn't match input",
+        text.substring(fileLen - 1024).getBytes(), out.toByteArray());
+    out.reset();
   }
 
   /**
@@ -915,75 +876,56 @@ public class TestDFSShell {
    */
   @Test(timeout = 30000)
   public void testTailWithFresh() throws Exception {
-    final int blockSize = 1024;
-    final Configuration conf = new Configuration();
-    conf.setInt(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, blockSize);
+    final Path testFile = new Path("testTailWithFresh", "file1");
+    dfs.create(testFile);
 
-    try (MiniDFSCluster cluster =
-             new MiniDFSCluster.Builder(conf).numDataNodes(3).build()) {
-      cluster.waitActive();
-      final DistributedFileSystem dfs = cluster.getFileSystem();
-      final Path testFile = new Path("testTailWithFresh", "file1");
-      dfs.create(testFile);
-
-      final ByteArrayOutputStream out = new ByteArrayOutputStream();
-      System.setOut(new PrintStream(out));
-      final Thread tailer = new Thread() {
-        @Override
-        public void run() {
-          final String[] argv = new String[]{"-tail", "-f",
-              testFile.toString()};
-          try {
-            ToolRunner.run(new FsShell(conf), argv);
-          } catch (Exception e) {
-            LOG.error("Client that tails the test file fails", e);
-          } finally {
-            out.reset();
-          }
-        }
-      };
-      tailer.start();
-      // wait till the tailer is sleeping
-      GenericTestUtils.waitFor(new Supplier<Boolean>() {
-        @Override
-        public Boolean get() {
-          return tailer.getState() == Thread.State.TIMED_WAITING;
+    final ByteArrayOutputStream out = new ByteArrayOutputStream();
+    System.setOut(new PrintStream(out));
+    final Thread tailer = new Thread() {
+      @Override
+      public void run() {
+        final String[] argv = new String[]{"-tail", "-f",
+            testFile.toString()};
+        try {
+          ToolRunner.run(new FsShell(dfs.getConf()), argv);
+        } catch (Exception e) {
+          LOG.error("Client that tails the test file fails", e);
+        } finally {
+          out.reset();
         }
-      }, 100, 10000);
-
-      final String text = RandomStringUtils.randomAscii(blockSize / 2);
-      try (OutputStream pout = dfs.create(testFile)) {
-        pout.write(text.getBytes());
       }
-      // The tailer should eventually show the file contents
-      GenericTestUtils.waitFor(new Supplier<Boolean>() {
-        @Override
-        public Boolean get() {
-          return Arrays.equals(text.getBytes(), out.toByteArray());
-        }
-      }, 100, 10000);
+    };
+    tailer.start();
+    // wait till the tailer is sleeping
+    GenericTestUtils.waitFor(new Supplier<Boolean>() {
+      @Override
+      public Boolean get() {
+        return tailer.getState() == Thread.State.TIMED_WAITING;
+      }
+    }, 100, 10000);
+
+    final String text = RandomStringUtils.randomAscii(BLOCK_SIZE / 2);
+    try (OutputStream pout = dfs.create(testFile)) {
+      pout.write(text.getBytes());
     }
+    // The tailer should eventually show the file contents
+    GenericTestUtils.waitFor(new Supplier<Boolean>() {
+      @Override
+      public Boolean get() {
+        return Arrays.equals(text.getBytes(), out.toByteArray());
+      }
+    }, 100, 10000);
   }
 
   @Test (timeout = 30000)
   public void testText() throws Exception {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = null;
-    try {
-      cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-      final FileSystem dfs = cluster.getFileSystem();
-      textTest(new Path("/texttest").makeQualified(dfs.getUri(),
-            dfs.getWorkingDirectory()), conf);
-
-      conf.set("fs.defaultFS", dfs.getUri().toString());
-      final FileSystem lfs = FileSystem.getLocal(conf);
-      textTest(new Path(TEST_ROOT_DIR, "texttest").makeQualified(lfs.getUri(),
-            lfs.getWorkingDirectory()), conf);
-    } finally {
-      if (null != cluster) {
-        cluster.shutdown();
-      }
-    }
+    final Configuration conf = dfs.getConf();
+    textTest(new Path("/texttest").makeQualified(dfs.getUri(),
+          dfs.getWorkingDirectory()), conf);
+
+    final FileSystem lfs = FileSystem.getLocal(conf);
+    textTest(new Path(TEST_ROOT_DIR, "texttest").makeQualified(lfs.getUri(),
+          lfs.getWorkingDirectory()), conf);
   }
 
   private void textTest(Path root, Configuration conf) throws Exception {
@@ -1082,75 +1024,60 @@ public class TestDFSShell {
 
   @Test (timeout = 30000)
   public void testCopyToLocal() throws IOException {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    FileSystem fs = cluster.getFileSystem();
-    assertTrue("Not a HDFS: "+fs.getUri(),
-               fs instanceof DistributedFileSystem);
-    DistributedFileSystem dfs = (DistributedFileSystem)fs;
-    FsShell shell = new FsShell();
-    shell.setConf(conf);
+    FsShell shell = new FsShell(dfs.getConf());
 
-    try {
-      String root = createTree(dfs, "copyToLocal");
+    String root = createTree(dfs, "copyToLocal");
 
-      // Verify copying the tree
-      {
-        try {
-          assertEquals(0,
-              runCmd(shell, "-copyToLocal", root + "*", TEST_ROOT_DIR));
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
+    // Verify copying the tree
+    {
+      try {
+        assertEquals(0,
+            runCmd(shell, "-copyToLocal", root + "*", TEST_ROOT_DIR));
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
 
-        File localroot = new File(TEST_ROOT_DIR, "copyToLocal");
-        File localroot2 = new File(TEST_ROOT_DIR, "copyToLocal2");        
-        
-        File f1 = new File(localroot, "f1");
-        assertTrue("Copying failed.", f1.isFile());
+      File localroot = new File(TEST_ROOT_DIR, "copyToLocal");
+      File localroot2 = new File(TEST_ROOT_DIR, "copyToLocal2");
 
-        File f2 = new File(localroot, "f2");
-        assertTrue("Copying failed.", f2.isFile());
+      File f1 = new File(localroot, "f1");
+      assertTrue("Copying failed.", f1.isFile());
 
-        File sub = new File(localroot, "sub");
-        assertTrue("Copying failed.", sub.isDirectory());
+      File f2 = new File(localroot, "f2");
+      assertTrue("Copying failed.", f2.isFile());
 
-        File f3 = new File(sub, "f3");
-        assertTrue("Copying failed.", f3.isFile());
+      File sub = new File(localroot, "sub");
+      assertTrue("Copying failed.", sub.isDirectory());
 
-        File f4 = new File(sub, "f4");
-        assertTrue("Copying failed.", f4.isFile());
-        
-        File f5 = new File(localroot2, "f1");
-        assertTrue("Copying failed.", f5.isFile());        
+      File f3 = new File(sub, "f3");
+      assertTrue("Copying failed.", f3.isFile());
 
-        f1.delete();
-        f2.delete();
-        f3.delete();
-        f4.delete();
-        f5.delete();
-        sub.delete();
-      }
-      // Verify copying non existing sources do not create zero byte
-      // destination files
-      {
-        String[] args = {"-copyToLocal", "nosuchfile", TEST_ROOT_DIR};
-        try {   
-          assertEquals(1, shell.run(args));
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                            e.getLocalizedMessage());
-        }                            
-        File f6 = new File(TEST_ROOT_DIR, "nosuchfile");
-        assertTrue(!f6.exists());
-      }
-    } finally {
+      File f4 = new File(sub, "f4");
+      assertTrue("Copying failed.", f4.isFile());
+
+      File f5 = new File(localroot2, "f1");
+      assertTrue("Copying failed.", f5.isFile());
+
+      f1.delete();
+      f2.delete();
+      f3.delete();
+      f4.delete();
+      f5.delete();
+      sub.delete();
+    }
+    // Verify copying non existing sources do not create zero byte
+    // destination files
+    {
+      String[] args = {"-copyToLocal", "nosuchfile", TEST_ROOT_DIR};
       try {
-        dfs.close();
+        assertEquals(1, shell.run(args));
       } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                          e.getLocalizedMessage());
       }
-      cluster.shutdown();
+      File f6 = new File(TEST_ROOT_DIR, "nosuchfile");
+      assertTrue(!f6.exists());
     }
   }
 
@@ -1180,63 +1107,42 @@ public class TestDFSShell {
 
   @Test (timeout = 30000)
   public void testCount() throws Exception {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    DistributedFileSystem dfs = cluster.getFileSystem();
-    FsShell shell = new FsShell();
-    shell.setConf(conf);
+    FsShell shell = new FsShell(dfs.getConf());
 
-    try {
-      String root = createTree(dfs, "count");
-
-      // Verify the counts
-      runCount(root, 2, 4, shell);
-      runCount(root + "2", 2, 1, shell);
-      runCount(root + "2/f1", 0, 1, shell);
-      runCount(root + "2/sub", 1, 0, shell);
-
-      final FileSystem localfs = FileSystem.getLocal(conf);
-      Path localpath = new Path(TEST_ROOT_DIR, "testcount");
-      localpath = localpath.makeQualified(localfs.getUri(),
-          localfs.getWorkingDirectory());
-      localfs.mkdirs(localpath);
-      
-      final String localstr = localpath.toString();
-      System.out.println("localstr=" + localstr);
-      runCount(localstr, 1, 0, shell);
-      assertEquals(0, runCmd(shell, "-count", root, localstr));
-    } finally {
-      try {
-        dfs.close();
-      } catch (Exception e) {
-      }
-      cluster.shutdown();
-    }
+    String root = createTree(dfs, "count");
+
+    // Verify the counts
+    runCount(root, 2, 4, shell);
+    runCount(root + "2", 2, 1, shell);
+    runCount(root + "2/f1", 0, 1, shell);
+    runCount(root + "2/sub", 1, 0, shell);
+
+    final FileSystem localfs = FileSystem.getLocal(dfs.getConf());
+    Path localpath = new Path(TEST_ROOT_DIR, "testcount");
+    localpath = localpath.makeQualified(localfs.getUri(),
+        localfs.getWorkingDirectory());
+    localfs.mkdirs(localpath);
+
+    final String localstr = localpath.toString();
+    System.out.println("localstr=" + localstr);
+    runCount(localstr, 1, 0, shell);
+    assertEquals(0, runCmd(shell, "-count", root, localstr));
   }
 
   @Test(timeout = 30000)
   public void testTotalSizeOfAllFiles() throws Exception {
-    Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = null;
-    try {
-      cluster = new MiniDFSCluster.Builder(conf).numDataNodes(1).build();
-      FileSystem fs = cluster.getFileSystem();
-      // create file under root
-      FSDataOutputStream File1 = fs.create(new Path("/File1"));
-      File1.write("hi".getBytes());
-      File1.close();
-      // create file under sub-folder
-      FSDataOutputStream File2 = fs.create(new Path("/Folder1/File2"));
-      File2.write("hi".getBytes());
-      File2.close();
-      // getUsed() should return total length of all the files in Filesystem
-      assertEquals(4, fs.getUsed());
-    } finally {
-      if (cluster != null) {
-        cluster.shutdown();
-        cluster = null;
-      }
-    }
+    final Path root = new Path("/testTotalSizeOfAllFiles");
+    dfs.mkdirs(root);
+    // create file under root
+    FSDataOutputStream File1 = dfs.create(new Path(root, "File1"));
+    File1.write("hi".getBytes());
+    File1.close();
+    // create file under sub-folder
+    FSDataOutputStream File2 = dfs.create(new Path(root, "Folder1/File2"));
+    File2.write("hi".getBytes());
+    File2.close();
+    // getUsed() should return total length of all the files in Filesystem
+    assertEquals(4, dfs.getUsed(root));
   }
 
   private static void runCount(String path, long dirs, long files, FsShell shell
@@ -1302,7 +1208,7 @@ public class TestDFSShell {
 
       //create an empty file
       Path file = new Path(chmodDir, "file");
-      TestDFSShell.writeFile(fs, file);
+      writeFile(fs, file);
 
       //test octal mode
       confirmPermissionChange("644", "rw-r--r--", fs, shell, file);
@@ -1347,7 +1253,6 @@ public class TestDFSShell {
 
     } finally {
       try {
-        fs.close();
         shell.close();
       } catch (IOException ignored) {}
     }
@@ -1390,16 +1295,14 @@ public class TestDFSShell {
     conf.set(DFSConfigKeys.DFS_PERMISSIONS_ENABLED_KEY, "true");
     
     //test chmod on DFS
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    fs = cluster.getFileSystem();
+    fs = dfs;
+    conf = dfs.getConf();
     testChmod(conf, fs, "/tmp/chmodTest");
     
     // test chown and chgrp on DFS:
     
     FsShell shell = new FsShell();
     shell.setConf(conf);
-    fs = cluster.getFileSystem();
-    
     /* For dfs, I am the super user and I can change owner of any file to
      * anything. "-R" option is already tested by chmod test above.
      */
@@ -1408,7 +1311,7 @@ public class TestDFSShell {
     Path path = new Path(file);
     Path parent = new Path("/tmp");
     Path root = new Path("/");
-    TestDFSShell.writeFile(fs, path);
+    writeFile(fs, path);
     
     runCmd(shell, "-chgrp", "-R", "herbivores", "/*", "unknownFile*");
     confirmOwner(null, "herbivores", fs, parent, path);
@@ -1438,439 +1341,423 @@ public class TestDFSShell {
     
     runCmd(shell, "-chgrp", "hadoop-core@apache.org/100", file);
     confirmOwner(null, "hadoop-core@apache.org/100", fs, path);
-    
-    cluster.shutdown();
   }
+
   /**
    * Tests various options of DFSShell.
    */
   @Test (timeout = 120000)
   public void testDFSShell() throws Exception {
-    final Configuration conf = new HdfsConfiguration();
     /* This tests some properties of ChecksumFileSystem as well.
      * Make sure that we create ChecksumDFS */
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    FileSystem fs = cluster.getFileSystem();
-    assertTrue("Not a HDFS: "+fs.getUri(),
-            fs instanceof DistributedFileSystem);
-    DistributedFileSystem fileSys = (DistributedFileSystem)fs;
-    FsShell shell = new FsShell();
-    shell.setConf(conf);
+    FsShell shell = new FsShell(dfs.getConf());
+
+    // First create a new directory with mkdirs
+    Path myPath = new Path("/test/mkdirs");
+    assertTrue(dfs.mkdirs(myPath));
+    assertTrue(dfs.exists(myPath));
+    assertTrue(dfs.mkdirs(myPath));
+
+    // Second, create a file in that directory.
+    Path myFile = new Path("/test/mkdirs/myFile");
+    writeFile(dfs, myFile);
+    assertTrue(dfs.exists(myFile));
+    Path myFile2 = new Path("/test/mkdirs/myFile2");
+    writeFile(dfs, myFile2);
+    assertTrue(dfs.exists(myFile2));
+
+    // Verify that rm with a pattern
+    {
+      String[] args = new String[2];
+      args[0] = "-rm";
+      args[1] = "/test/mkdirs/myFile*";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertTrue(val == 0);
+      assertFalse(dfs.exists(myFile));
+      assertFalse(dfs.exists(myFile2));
+
+      //re-create the files for other tests
+      writeFile(dfs, myFile);
+      assertTrue(dfs.exists(myFile));
+      writeFile(dfs, myFile2);
+      assertTrue(dfs.exists(myFile2));
+    }
 
-    try {
-      // First create a new directory with mkdirs
-      Path myPath = new Path("/test/mkdirs");
-      assertTrue(fileSys.mkdirs(myPath));
-      assertTrue(fileSys.exists(myPath));
-      assertTrue(fileSys.mkdirs(myPath));
-
-      // Second, create a file in that directory.
-      Path myFile = new Path("/test/mkdirs/myFile");
-      writeFile(fileSys, myFile);
-      assertTrue(fileSys.exists(myFile));
-      Path myFile2 = new Path("/test/mkdirs/myFile2");      
-      writeFile(fileSys, myFile2);
-      assertTrue(fileSys.exists(myFile2));
-
-      // Verify that rm with a pattern
-      {
-        String[] args = new String[2];
-        args[0] = "-rm";
-        args[1] = "/test/mkdirs/myFile*";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage()); 
-        }
-        assertTrue(val == 0);
-        assertFalse(fileSys.exists(myFile));
-        assertFalse(fileSys.exists(myFile2));
+    // Verify that we can read the file
+    {
+      String[] args = new String[3];
+      args[0] = "-cat";
+      args[1] = "/test/mkdirs/myFile";
+      args[2] = "/test/mkdirs/myFile2";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run: " +
+                           StringUtils.stringifyException(e));
+      }
+      assertTrue(val == 0);
+    }
+    dfs.delete(myFile2, true);
 
-        //re-create the files for other tests
-        writeFile(fileSys, myFile);
-        assertTrue(fileSys.exists(myFile));
-        writeFile(fileSys, myFile2);
-        assertTrue(fileSys.exists(myFile2));
+    // Verify that we get an error while trying to read an nonexistent file
+    {
+      String[] args = new String[2];
+      args[0] = "-cat";
+      args[1] = "/test/mkdirs/myFile1";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertTrue(val != 0);
+    }
 
-      // Verify that we can read the file
-      {
-        String[] args = new String[3];
-        args[0] = "-cat";
-        args[1] = "/test/mkdirs/myFile";
-        args[2] = "/test/mkdirs/myFile2";        
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run: " +
-                             StringUtils.stringifyException(e)); 
-        }
-        assertTrue(val == 0);
+    // Verify that we get an error while trying to delete an nonexistent file
+    {
+      String[] args = new String[2];
+      args[0] = "-rm";
+      args[1] = "/test/mkdirs/myFile1";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
-      fileSys.delete(myFile2, true);
+      assertTrue(val != 0);
+    }
 
-      // Verify that we get an error while trying to read an nonexistent file
-      {
-        String[] args = new String[2];
-        args[0] = "-cat";
-        args[1] = "/test/mkdirs/myFile1";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage()); 
-        }
-        assertTrue(val != 0);
+    // Verify that we succeed in removing the file we created
+    {
+      String[] args = new String[2];
+      args[0] = "-rm";
+      args[1] = "/test/mkdirs/myFile";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertTrue(val == 0);
+    }
 
-      // Verify that we get an error while trying to delete an nonexistent file
-      {
-        String[] args = new String[2];
-        args[0] = "-rm";
-        args[1] = "/test/mkdirs/myFile1";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage()); 
-        }
-        assertTrue(val != 0);
+    // Verify touch/test
+    {
+      String[] args;
+      int val;
+
+      args = new String[3];
+      args[0] = "-test";
+      args[1] = "-e";
+      args[2] = "/test/mkdirs/noFileHere";
+      val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(1, val);
 
-      // Verify that we succeed in removing the file we created
-      {
-        String[] args = new String[2];
-        args[0] = "-rm";
-        args[1] = "/test/mkdirs/myFile";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage()); 
-        }
-        assertTrue(val == 0);
+      args[1] = "-z";
+      val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(1, val);
 
-      // Verify touch/test
-      {
-        String[] args;
-        int val;
-
-        args = new String[3];
-        args[0] = "-test";
-        args[1] = "-e";
-        args[2] = "/test/mkdirs/noFileHere";
-        val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+      args = new String[2];
+      args[0] = "-touchz";
+      args[1] = "/test/mkdirs/isFileHere";
+      val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertEquals(0, val);
 
-        args[1] = "-z";
-        val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+      args = new String[2];
+      args[0] = "-touchz";
+      args[1] = "/test/mkdirs/thisDirNotExists/isFileHere";
+      val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertEquals(1, val);
 
-        args = new String[2];
-        args[0] = "-touchz";
-        args[1] = "/test/mkdirs/isFileHere";
-        val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
 
-        args = new String[2];
-        args[0] = "-touchz";
-        args[1] = "/test/mkdirs/thisDirNotExists/isFileHere";
-        val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+      args = new String[3];
+      args[0] = "-test";
+      args[1] = "-e";
+      args[2] = "/test/mkdirs/isFileHere";
+      val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertEquals(0, val);
 
+      args[1] = "-d";
+      val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertEquals(1, val);
 
-        args = new String[3];
-        args[0] = "-test";
-        args[1] = "-e";
-        args[2] = "/test/mkdirs/isFileHere";
-        val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
+      args[1] = "-z";
+      val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertEquals(0, val);
+    }
 
-        args[1] = "-d";
-        val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+    // Verify that cp from a directory to a subdirectory fails
+    {
+      String[] args = new String[2];
+      args[0] = "-mkdir";
+      args[1] = "/test/dir1";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertEquals(0, val);
 
-        args[1] = "-z";
-        val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
+      // this should fail
+      String[] args1 = new String[3];
+      args1[0] = "-cp";
+      args1[1] = "/test/dir1";
+      args1[2] = "/test/dir1/dir2";
+      val = 0;
+      try {
+        val = shell.run(args1);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(1, val);
 
-      // Verify that cp from a directory to a subdirectory fails
-      {
-        String[] args = new String[2];
-        args[0] = "-mkdir";
-        args[1] = "/test/dir1";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
-
-        // this should fail
-        String[] args1 = new String[3];
-        args1[0] = "-cp";
-        args1[1] = "/test/dir1";
-        args1[2] = "/test/dir1/dir2";
-        val = 0;
-        try {
-          val = shell.run(args1);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+      // this should succeed
+      args1[0] = "-cp";
+      args1[1] = "/test/dir1";
+      args1[2] = "/test/dir1foo";
+      val = -1;
+      try {
+        val = shell.run(args1);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
+      }
+      assertEquals(0, val);
 
-        // this should succeed
-        args1[0] = "-cp";
-        args1[1] = "/test/dir1";
-        args1[2] = "/test/dir1foo";
-        val = -1;
-        try {
-          val = shell.run(args1);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
+      // this should fail
+      args1[0] = "-cp";
+      args1[1] = "/";
+      args1[2] = "/test";
+      val = 0;
+      try {
+        val = shell.run(args1);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+            e.getLocalizedMessage());
+      }
+      assertEquals(1, val);
+    }
 
-        // this should fail
-        args1[0] = "-cp";
-        args1[1] = "/";
-        args1[2] = "/test";
-        val = 0;
-        try {
-          val = shell.run(args1);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-              e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+    // Verify -test -f negative case (missing file)
+    {
+      String[] args = new String[3];
+      args[0] = "-test";
+      args[1] = "-f";
+      args[2] = "/test/mkdirs/noFileHere";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(1, val);
+    }
 
-      // Verify -test -f negative case (missing file)
-      {
-        String[] args = new String[3];
-        args[0] = "-test";
-        args[1] = "-f";
-        args[2] = "/test/mkdirs/noFileHere";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+    // Verify -test -f negative case (directory rather than file)
+    {
+      String[] args = new String[3];
+      args[0] = "-test";
+      args[1] = "-f";
+      args[2] = "/test/mkdirs";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(1, val);
+    }
 
-      // Verify -test -f negative case (directory rather than file)
-      {
-        String[] args = new String[3];
-        args[0] = "-test";
-        args[1] = "-f";
-        args[2] = "/test/mkdirs";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
-      }
+    // Verify -test -f positive case
+    {
+      writeFile(dfs, myFile);
+      assertTrue(dfs.exists(myFile));
 
-      // Verify -test -f positive case
-      {
-        writeFile(fileSys, myFile);
-        assertTrue(fileSys.exists(myFile));
-
-        String[] args = new String[3];
-        args[0] = "-test";
-        args[1] = "-f";
-        args[2] = myFile.toString();
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
+      String[] args = new String[3];
+      args[0] = "-test";
+      args[1] = "-f";
+      args[2] = myFile.toString();
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(0, val);
+    }
 
-      // Verify -test -s negative case (missing file)
-      {
-        String[] args = new String[3];
-        args[0] = "-test";
-        args[1] = "-s";
-        args[2] = "/test/mkdirs/noFileHere";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+    // Verify -test -s negative case (missing file)
+    {
+      String[] args = new String[3];
+      args[0] = "-test";
+      args[1] = "-s";
+      args[2] = "/test/mkdirs/noFileHere";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(1, val);
+    }
 
-      // Verify -test -s negative case (zero length file)
-      {
-        String[] args = new String[3];
-        args[0] = "-test";
-        args[1] = "-s";
-        args[2] = "/test/mkdirs/isFileHere";
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(1, val);
+    // Verify -test -s negative case (zero length file)
+    {
+      String[] args = new String[3];
+      args[0] = "-test";
+      args[1] = "-s";
+      args[2] = "/test/mkdirs/isFileHere";
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(1, val);
+    }
 
-      // Verify -test -s positive case (nonzero length file)
-      {
-        String[] args = new String[3];
-        args[0] = "-test";
-        args[1] = "-s";
-        args[2] = myFile.toString();
-        int val = -1;
-        try {
-          val = shell.run(args);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-                             e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
+    // Verify -test -s positive case (nonzero length file)
+    {
+      String[] args = new String[3];
+      args[0] = "-test";
+      args[1] = "-s";
+      args[2] = myFile.toString();
+      int val = -1;
+      try {
+        val = shell.run(args);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+                           e.getLocalizedMessage());
       }
+      assertEquals(0, val);
+    }
 
-      // Verify -test -w/-r
-      {
-        Path permDir = new Path("/test/permDir");
-        Path permFile = new Path("/test/permDir/permFile");
-        mkdir(fs, permDir);
-        writeFile(fs, permFile);
-
-        // Verify -test -w positive case (dir exists and can write)
-        final String[] wargs = new String[3];
-        wargs[0] = "-test";
-        wargs[1] = "-w";
-        wargs[2] = permDir.toString();
-        int val = -1;
-        try {
-          val = shell.run(wargs);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-              e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
+    // Verify -test -w/-r
+    {
+      Path permDir = new Path("/test/permDir");
+      Path permFile = new Path("/test/permDir/permFile");
+      mkdir(dfs, permDir);
+      writeFile(dfs, permFile);
+
+      // Verify -test -w positive case (dir exists and can write)
+      final String[] wargs = new String[3];
+      wargs[0] = "-test";
+      wargs[1] = "-w";
+      wargs[2] = permDir.toString();
+      int val = -1;
+      try {
+        val = shell.run(wargs);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+            e.getLocalizedMessage());
+      }
+      assertEquals(0, val);
 
-        // Verify -test -r positive case (file exists and can read)
-        final String[] rargs = new String[3];
-        rargs[0] = "-test";
-        rargs[1] = "-r";
-        rargs[2] = permFile.toString();
-        try {
-          val = shell.run(rargs);
-        } catch (Exception e) {
-          System.err.println("Exception raised from DFSShell.run " +
-              e.getLocalizedMessage());
-        }
-        assertEquals(0, val);
-
-        // Verify -test -r negative case (file exists but cannot read)
-        runCmd(shell, "-chmod", "600", permFile.toString());
-
-        UserGroupInformation smokeUser =
-            UserGroupInformation.createUserForTesting("smokeUser",
-                new String[] {"hadoop"});
-        smokeUser.doAs(new PrivilegedExceptionAction<String>() {
-            @Override
-            public String run() throws Exception {
-              FsShell shell = new FsShell(conf);
-              int exitCode = shell.run(rargs);
-              assertEquals(1, exitCode);
-              return null;
-            }
-          });
+      // Verify -test -r positive case (file exists and can read)
+      final String[] rargs = new String[3];
+      rargs[0] = "-test";
+      rargs[1] = "-r";
+      rargs[2] = permFile.toString();
+      try {
+        val = shell.run(rargs);
+      } catch (Exception e) {
+        System.err.println("Exception raised from DFSShell.run " +
+            e.getLocalizedMessage());
+      }
+      assertEquals(0, val);
 
-        // Verify -test -w negative case (dir exists but cannot write)
-        runCmd(shell, "-chown", "-R", "not_allowed", permDir.toString());
-        runCmd(shell, "-chmod", "-R", "700", permDir.toString());
+      // Verify -test -r negative case (file exists but cannot read)
+      runCmd(shell, "-chmod", "600", permFile.toString());
 
-        smokeUser.doAs(new PrivilegedExceptionAction<String>() {
+      UserGroupInformation smokeUser =
+          UserGroupInformation.createUserForTesting("smokeUser",
+              new String[] {"hadoop"});
+      smokeUser.doAs(new PrivilegedExceptionAction<String>() {
           @Override
           public String run() throws Exception {
-            FsShell shell = new FsShell(conf);
-            int exitCode = shell.run(wargs);
+            FsShell shell = new FsShell(dfs.getConf());
+            int exitCode = shell.run(rargs);
             assertEquals(1, exitCode);
             return null;
           }
         });
 
-        // cleanup
-        fs.delete(permDir, true);
-      }
-    } finally {
-      try {
-        fileSys.close();
-      } catch (Exception e) {
-      }
-      cluster.shutdown();
+      // Verify -test -w negative case (dir exists but cannot write)
+      runCmd(shell, "-chown", "-R", "not_allowed", permDir.toString());
+      runCmd(shell, "-chmod", "-R", "700", permDir.toString());
+
+      smokeUser.doAs(new PrivilegedExceptionAction<String>() {
+        @Override
+        public String run() throws Exception {
+          FsShell shell = new FsShell(dfs.getConf());
+          int exitCode = shell.run(wargs);
+          assertEquals(1, exitCode);
+          return null;
+        }
+      });
+
+      // cleanup
+      dfs.delete(permDir, true);
     }
   }
 
@@ -1910,21 +1797,17 @@ public class TestDFSShell {
   public void testRemoteException() throws Exception {
     UserGroupInformation tmpUGI = 
       UserGroupInformation.createUserForTesting("tmpname", new String[] {"mygroup"});
-    MiniDFSCluster dfs = null;
     PrintStream bak = null;
     try {
-      final Configuration conf = new HdfsConfiguration();
-      dfs = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-      FileSystem fs = dfs.getFileSystem();
       Path p = new Path("/foo");
-      fs.mkdirs(p);
-      fs.setPermission(p, new FsPermission((short)0700));
+      dfs.mkdirs(p);
+      dfs.setPermission(p, new FsPermission((short)0700));
       bak = System.err;
       
       tmpUGI.doAs(new PrivilegedExceptionAction<Object>() {
         @Override
         public Object run() throws Exception {
-          FsShell fshell = new FsShell(conf);
+          FsShell fshell = new FsShell(dfs.getConf());
           ByteArrayOutputStream out = new ByteArrayOutputStream();
           PrintStream tmp = new PrintStream(out);
           System.setErr(tmp);
@@ -1944,9 +1827,6 @@ public class TestDFSShell {
       if (bak != null) {
         System.setErr(bak);
       }
-      if (dfs != null) {
-        dfs.shutdown();
-      }
     }
   }
   
@@ -2004,13 +1884,13 @@ public class TestDFSShell {
       // find block files to modify later
       List<MaterializedReplica> replicas = getMaterializedReplicas(cluster);
 
-      // Shut down cluster and then corrupt the block files by overwriting a
-      // portion with junk data.  We must shut down the cluster so that threads
+      // Shut down miniCluster and then corrupt the block files by overwriting a
+      // portion with junk data.  We must shut down the miniCluster so that threads
       // in the data node do not hold locks on the block files while we try to
       // write into them.  Particularly on Windows, the data node's use of the
       // FileChannel.transferTo method can cause block files to be memory mapped
       // in read-only mode during the transfer to a client, and this causes a
-      // locking conflict.  The call to shutdown the cluster blocks until all
+      // locking conflict.  The call to shutdown the miniCluster blocks until all
       // DataXceiver threads exit, preventing this problem.
       dfs.close();
       cluster.shutdown();
@@ -2018,7 +1898,7 @@ public class TestDFSShell {
       show("replicas=" + replicas);
       corrupt(replicas, localfcontent);
 
-      // Start the cluster again, but do not reformat, so prior files remain.
+      // Start the miniCluster again, but do not reformat, so prior files remain.
       cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).format(false)
         .build();
       dfs = cluster.getFileSystem();
@@ -2047,55 +1927,45 @@ public class TestDFSShell {
    */
   @Test (timeout = 30000)
   public void testStat() throws Exception {
-    final int blockSize = 1024;
-    final Configuration conf = new HdfsConfiguration();
-    conf.setInt(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, blockSize);
-
-    try (MiniDFSCluster cluster =
-             new MiniDFSCluster.Builder(conf).numDataNodes(3).build()) {
-      cluster.waitActive();
-      final DistributedFileSystem dfs = cluster.getFileSystem();
-
-      final SimpleDateFormat fmt = new SimpleDateFormat("yyyy-MM-dd HH:mm:ss");
-      fmt.setTimeZone(TimeZone.getTimeZone("UTC"));
-      final Path testDir1 = new Path("testStat", "dir1");
-      dfs.mkdirs(testDir1);
-      final Path testFile2 = new Path(testDir1, "file2");
-      DFSTestUtil.createFile(dfs, testFile2, 2 * blockSize, (short) 3, 0);
-      final FileStatus status1 = dfs.getFileStatus(testDir1);
-      final String mtime1 = fmt.format(new Date(status1.getModificationTime()));
-      final FileStatus status2 = dfs.getFileStatus(testFile2);
-      final String mtime2 = fmt.format(new Date(status2.getModificationTime()));
+    final SimpleDateFormat fmt = new SimpleDateFormat("yyyy-MM-dd HH:mm:ss");
+    fmt.setTimeZone(TimeZone.getTimeZone("UTC"));
+    final Path testDir1 = new Path("testStat", "dir1");
+    dfs.mkdirs(testDir1);
+    final Path testFile2 = new Path(testDir1, "file2");
+    DFSTestUtil.createFile(dfs, testFile2, 2 * BLOCK_SIZE, (short) 3, 0);
+    final FileStatus status1 = dfs.getFileStatus(testDir1);
+    final String mtime1 = fmt.format(new Date(status1.getModificationTime()));
+    final FileStatus status2 = dfs.getFileStatus(testFile2);
+    final String mtime2 = fmt.format(new Date(status2.getModificationTime()));
 
-      final ByteArrayOutputStream out = new ByteArrayOutputStream();
-      System.setOut(new PrintStream(out));
+    final ByteArrayOutputStream out = new ByteArrayOutputStream();
+    System.setOut(new PrintStream(out));
 
-      doFsStat(conf, null);
+    doFsStat(dfs.getConf(), null);
 
-      out.reset();
-      doFsStat(conf, null, testDir1);
-      assertEquals("Unexpected -stat output: " + out,
-          out.toString(), String.format("%s%n", mtime1));
+    out.reset();
+    doFsStat(dfs.getConf(), null, testDir1);
+    assertEquals("Unexpected -stat output: " + out,
+        out.toString(), String.format("%s%n", mtime1));
 
-      out.reset();
-      doFsStat(conf, null, testDir1, testFile2);
-      assertEquals("Unexpected -stat output: " + out,
-          out.toString(), String.format("%s%n%s%n", mtime1, mtime2));
+    out.reset();
+    doFsStat(dfs.getConf(), null, testDir1, testFile2);
+    assertEquals("Unexpected -stat output: " + out,
+        out.toString(), String.format("%s%n%s%n", mtime1, mtime2));
 
-      doFsStat(conf, "%F %u:%g %b %y %n");
+    doFsStat(dfs.getConf(), "%F %u:%g %b %y %n");
 
-      out.reset();
-      doFsStat(conf, "%F %u:%g %b %y %n", testDir1);
-      assertTrue(out.toString(), out.toString().contains(mtime1));
-      assertTrue(out.toString(), out.toString().contains("directory"));
-      assertTrue(out.toString(), out.toString().contains(status1.getGroup()));
+    out.reset();
+    doFsStat(dfs.getConf(), "%F %u:%g %b %y %n", testDir1);
+    assertTrue(out.toString(), out.toString().contains(mtime1));
+    assertTrue(out.toString(), out.toString().contains("directory"));
+    assertTrue(out.toString(), out.toString().contains(status1.getGroup()));
 
-      out.reset();
-      doFsStat(conf, "%F %u:%g %b %y %n", testDir1, testFile2);
-      assertTrue(out.toString(), out.toString().contains(mtime1));
-      assertTrue(out.toString(), out.toString().contains("regular file"));
-      assertTrue(out.toString(), out.toString().contains(mtime2));
-    }
+    out.reset();
+    doFsStat(dfs.getConf(), "%F %u:%g %b %y %n", testDir1, testFile2);
+    assertTrue(out.toString(), out.toString().contains(mtime1));
+    assertTrue(out.toString(), out.toString().contains("regular file"));
+    assertTrue(out.toString(), out.toString().contains(mtime2));
   }
 
   private static void doFsStat(Configuration conf, String format, Path... files)
@@ -2122,33 +1992,26 @@ public class TestDFSShell {
 
   @Test (timeout = 30000)
   public void testLsr() throws Exception {
-    final Configuration conf = new HdfsConfiguration();
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(2).build();
-    DistributedFileSystem dfs = cluster.getFileSystem();
+    final Configuration conf = dfs.getConf();
+    final String root = createTree(dfs, "lsr");
+    dfs.mkdirs(new Path(root, "zzz"));
 
-    try {
-      final String root = createTree(dfs, "lsr");
-      dfs.mkdirs(new Path(root, "zzz"));
-      
-      runLsr(new FsShell(conf), root, 0);
-      
-      final Path sub = new Path(root, "sub");
-      dfs.setPermission(sub, new FsPermission((short)0));
-
-      final UserGroupInformation ugi = UserGroupInformation.getCurrentUser();
-      final String tmpusername = ugi.getShortUserName() + "1";
-      UserGroupInformation tmpUGI = UserGroupInformation.createUserForTesting(
-          tmpusername, new String[] {tmpusername});
-      String results = tmpUGI.doAs(new PrivilegedExceptionAction<String>() {
-        @Override
-        public String run() throws Exception {
-          return runLsr(new FsShell(conf), root, 1);
-        }
-      });
-      assertTrue(results.contains("zzz"));
-    } finally {
-      cluster.shutdown();
-    }
+    runLsr(new FsShell(conf), root, 0);
+
+    final Path sub = new Path(root, "sub");
+    dfs.setPermission(sub, new FsPermission((short)0));
+
+    final UserGroupInformation ugi = UserGroupInformation.getCurrentUser();
+    final String tmpusername = ugi.getShortUserName() + "1";
+    UserGroupInformation tmpUGI = UserGroupInformation.createUserForTesting(
+        tmpusername, new String[] {tmpusername});
+    String results = tmpUGI.doAs(new PrivilegedExceptionAction<String>() {
+      @Override
+      public String run() throws Exception {
+        return runLsr(new FsShell(conf), root, 1);
+      }
+    });
+    assertTrue(results.contains("zzz"));
   }
 
   private static String runLsr(final FsShell shell, String root, int returnvalue
@@ -2192,40 +2055,33 @@ public class TestDFSShell {
   // ACLs)
   @Test (timeout = 120000)
   public void testCopyCommandsWithPreserveOption() throws Exception {
-    Configuration conf = new Configuration();
-    conf.setBoolean(DFSConfigKeys.DFS_NAMENODE_XATTRS_ENABLED_KEY, true);
-    conf.setBoolean(DFSConfigKeys.DFS_NAMENODE_ACLS_ENABLED_KEY, true);
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(1)
-        .format(true).build();
     FsShell shell = null;
-    FileSystem fs = null;
     final String testdir = "/tmp/TestDFSShell-testCopyCommandsWithPreserveOption-"
         + counter.getAndIncrement();
     final Path hdfsTestDir = new Path(testdir);
     try {
-      fs = cluster.getFileSystem();
-      fs.mkdirs(hdfsTestDir);
+      dfs.mkdirs(hdfsTestDir);
       Path src = new Path(hdfsTestDir, "srcfile");
-      fs.create(src).close();
+      dfs.create(src).close();
 
-      fs.setAcl(src, Lists.newArrayList(
+      dfs.setAcl(src, Lists.newArrayList(
           aclEntry(ACCESS, USER, ALL),
           aclEntry(ACCESS, USER, "foo", ALL),
           aclEntry(ACCESS, GROUP, READ_EXECUTE),
           aclEntry(ACCESS, GROUP, "bar", READ_EXECUTE),
           aclEntry(ACCESS, OTHER, EXECUTE)));
 
-      FileStatus status = fs.getFileStatus(src);
+      FileStatus status = dfs.getFileStatus(src);
       final long mtime = status.getModificationTime();
       final long atime = status.getAccessTime();
       final String owner = status.getOwner();
       final String group = status.getGroup();
       final FsPermission perm = status.getPermission();
+
+      dfs.setXAttr(src, USER_A1, USER_A1_VALUE);
+      dfs.setXAttr(src, TRUSTED_A1, TRUSTED_A1_VALUE);
       
-      fs.setXAttr(src, USER_A1, USER_A1_VALUE);
-      fs.setXAttr(src, TRUSTED_A1, TRUSTED_A1_VALUE);
-      
-      shell = new FsShell(conf);
+      shell = new FsShell(dfs.getConf());
       
       // -p
       Path target1 = new Path(hdfsTestDir, "targetfile1");
@@ -2233,16 +2089,16 @@ public class TestDFSShell {
           target1.toUri().toString() };
       int ret = ToolRunner.run(shell, argv);
       assertEquals("cp -p is not working", SUCCESS, ret);
-      FileStatus targetStatus = fs.getFileStatus(target1);
+      FileStatus targetStatus = dfs.getFileStatus(target1);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       FsPermission targetPerm = targetStatus.getPermission();
       assertTrue(perm.equals(targetPerm));
-      Map<String, byte[]> xattrs = fs.getXAttrs(target1);
+      Map<String, byte[]> xattrs = dfs.getXAttrs(target1);
       assertTrue(xattrs.isEmpty());
-      List<AclEntry> acls = fs.getAclStatus(target1).getEntries();
+      List<AclEntry> acls = dfs.getAclStatus(target1).getEntries();
       assertTrue(acls.isEmpty());
       assertFalse(targetPerm.getAclBit());
 
@@ -2252,16 +2108,16 @@ public class TestDFSShell {
           target2.toUri().toString() };
       ret = ToolRunner.run(shell, argv);
       assertEquals("cp -ptop is not working", SUCCESS, ret);
-      targetStatus = fs.getFileStatus(target2);
+      targetStatus = dfs.getFileStatus(target2);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       targetPerm = targetStatus.getPermission();
       assertTrue(perm.equals(targetPerm));
-      xattrs = fs.getXAttrs(target2);
+      xattrs = dfs.getXAttrs(target2);
       assertTrue(xattrs.isEmpty());
-      acls = fs.getAclStatus(target2).getEntries();
+      acls = dfs.getAclStatus(target2).getEntries();
       assertTrue(acls.isEmpty());
       assertFalse(targetPerm.getAclBit());
 
@@ -2271,18 +2127,18 @@ public class TestDFSShell {
           target3.toUri().toString() };
       ret = ToolRunner.run(shell, argv);
       assertEquals("cp -ptopx is not working", SUCCESS, ret);
-      targetStatus = fs.getFileStatus(target3);
+      targetStatus = dfs.getFileStatus(target3);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       targetPerm = targetStatus.getPermission();
       assertTrue(perm.equals(targetPerm));
-      xattrs = fs.getXAttrs(target3);
+      xattrs = dfs.getXAttrs(target3);
       assertEquals(xattrs.size(), 2);
       assertArrayEquals(USER_A1_VALUE, xattrs.get(USER_A1));
       assertArrayEquals(TRUSTED_A1_VALUE, xattrs.get(TRUSTED_A1));
-      acls = fs.getAclStatus(target3).getEntries();
+      acls = dfs.getAclStatus(target3).getEntries();
       assertTrue(acls.isEmpty());
       assertFalse(targetPerm.getAclBit());
 
@@ -2292,19 +2148,19 @@ public class TestDFSShell {
           target4.toUri().toString() };
       ret = ToolRunner.run(shell, argv);
       assertEquals("cp -ptopa is not working", SUCCESS, ret);
-      targetStatus = fs.getFileStatus(target4);
+      targetStatus = dfs.getFileStatus(target4);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       targetPerm = targetStatus.getPermission();
       assertTrue(perm.equals(targetPerm));
-      xattrs = fs.getXAttrs(target4);
+      xattrs = dfs.getXAttrs(target4);
       assertTrue(xattrs.isEmpty());
-      acls = fs.getAclStatus(target4).getEntries();
+      acls = dfs.getAclStatus(target4).getEntries();
       assertFalse(acls.isEmpty());
       assertTrue(targetPerm.getAclBit());
-      assertEquals(fs.getAclStatus(src), fs.getAclStatus(target4));
+      assertEquals(dfs.getAclStatus(src), dfs.getAclStatus(target4));
 
       // -ptoa (verify -pa option will preserve permissions also)
       Path target5 = new Path(hdfsTestDir, "targetfile5");
@@ -2312,59 +2168,47 @@ public class TestDFSShell {
           target5.toUri().toString() };
       ret = ToolRunner.run(shell, argv);
       assertEquals("cp -ptoa is not working", SUCCESS, ret);
-      targetStatus = fs.getFileStatus(target5);
+      targetStatus = dfs.getFileStatus(target5);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       targetPerm = targetStatus.getPermission();
       assertTrue(perm.equals(targetPerm));
-      xattrs = fs.getXAttrs(target5);
+      xattrs = dfs.getXAttrs(target5);
       assertTrue(xattrs.isEmpty());
-      acls = fs.getAclStatus(target5).getEntries();
+      acls = dfs.getAclStatus(target5).getEntries();
       assertFalse(acls.isEmpty());
       assertTrue(targetPerm.getAclBit());
-      assertEquals(fs.getAclStatus(src), fs.getAclStatus(target5));
+      assertEquals(dfs.getAclStatus(src), dfs.getAclStatus(target5));
     } finally {
       if (null != shell) {
         shell.close();
       }
-
-      if (null != fs) {
-        fs.delete(hdfsTestDir, true);
-        fs.close();
-      }
-      cluster.shutdown();
     }
   }
 
   @Test (timeout = 120000)
   public void testCopyCommandsWithRawXAttrs() throws Exception {
-    final Configuration conf = new Configuration();
-    conf.setBoolean(DFSConfigKeys.DFS_NAMENODE_XATTRS_ENABLED_KEY, true);
-    final MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).
-      numDataNodes(1).format(true).build();
     FsShell shell = null;
-    FileSystem fs = null;
     final String testdir = "/tmp/TestDFSShell-testCopyCommandsWithRawXAttrs-"
       + counter.getAndIncrement();
     final Path hdfsTestDir = new Path(testdir);
     final Path rawHdfsTestDir = new Path("/.reserved/raw" + testdir);
     try {
-      fs = cluster.getFileSystem();
-      fs.mkdirs(hdfsTestDir);
+      dfs.mkdirs(hdfsTestDir);
       final Path src = new Path(hdfsTestDir, "srcfile");
       final String rawSrcBase = "/.reserved/raw" + testdir;
       final Path rawSrc = new Path(rawSrcBase, "srcfile");
-      fs.create(src).close();
+      dfs.create(src).close();
 
       final Path srcDir = new Path(hdfsTestDir, "srcdir");
       final Path rawSrcDir = new Path("/.reserved/raw" + testdir, "srcdir");
-      fs.mkdirs(srcDir);
+      dfs.mkdirs(srcDir);
       final Path srcDirFile = new Path(srcDir, "srcfile");
       final Path rawSrcDirFile =
               new Path("/.reserved/raw" + srcDirFile);
-      fs.create(srcDirFile).close();
+      dfs.create(srcDirFile).close();
 
       final Path[] paths = { rawSrc, rawSrcDir, rawSrcDirFile };
       final String[] xattrNames = { USER_A1, RAW_A1 };
@@ -2372,35 +2216,35 @@ public class TestDFSShell {
 
       for (int i = 0; i < paths.length; i++) {
         for (int j = 0; j < xattrNames.length; j++) {
-          fs.setXAttr(paths[i], xattrNames[j], xattrVals[j]);
+          dfs.setXAttr(paths[i], xattrNames[j], xattrVals[j]);
         }
       }
 
-      shell = new FsShell(conf);
+      shell = new FsShell(dfs.getConf());
 
       /* Check that a file as the source path works ok. */
-      doTestCopyCommandsWithRawXAttrs(shell, fs, src, hdfsTestDir, false);
-      doTestCopyCommandsWithRawXAttrs(shell, fs, rawSrc, hdfsTestDir, false);
-      doTestCopyCommandsWithRawXAttrs(shell, fs, src, rawHdfsTestDir, false);
-      doTestCopyCommandsWithRawXAttrs(shell, fs, rawSrc, rawHdfsTestDir, true);
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, src, hdfsTestDir, false);
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, rawSrc, hdfsTestDir, false);
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, src, rawHdfsTestDir, false);
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, rawSrc, rawHdfsTestDir, true);
 
       /* Use a relative /.reserved/raw path. */
-      final Path savedWd = fs.getWorkingDirectory();
+      final Path savedWd = dfs.getWorkingDirectory();
       try {
-        fs.setWorkingDirectory(new Path(rawSrcBase));
+        dfs.setWorkingDirectory(new Path(rawSrcBase));
         final Path relRawSrc = new Path("../srcfile");
         final Path relRawHdfsTestDir = new Path("..");
-        doTestCopyCommandsWithRawXAttrs(shell, fs, relRawSrc, relRawHdfsTestDir,
-                true);
+        doTestCopyCommandsWithRawXAttrs(shell, dfs, relRawSrc,
+            relRawHdfsTestDir, true);
       } finally {
-        fs.setWorkingDirectory(savedWd);
+        dfs.setWorkingDirectory(savedWd);
       }
 
       /* Check that a directory as the source path works ok. */
-      doTestCopyCommandsWithRawXAttrs(shell, fs, srcDir, hdfsTestDir, false);
-      doTestCopyCommandsWithRawXAttrs(shell, fs, rawSrcDir, hdfsTestDir, false);
-      doTestCopyCommandsWithRawXAttrs(shell, fs, srcDir, rawHdfsTestDir, false);
-      doTestCopyCommandsWithRawXAttrs(shell, fs, rawSrcDir, rawHdfsTestDir,
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, srcDir, hdfsTestDir, false);
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, rawSrcDir, hdfsTestDir, false);
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, srcDir, rawHdfsTestDir, false);
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, rawSrcDir, rawHdfsTestDir,
         true);
 
       /* Use relative in an absolute path. */
@@ -2408,18 +2252,13 @@ public class TestDFSShell {
           testdir + "/srcdir";
       final String relRawDstDir = "./.reserved/../.reserved/raw/../raw" +
           testdir;
-      doTestCopyCommandsWithRawXAttrs(shell, fs, new Path(relRawSrcDir),
+      doTestCopyCommandsWithRawXAttrs(shell, dfs, new Path(relRawSrcDir),
           new Path(relRawDstDir), true);
     } finally {
       if (null != shell) {
         shell.close();
       }
-
-      if (null != fs) {
-        fs.delete(hdfsTestDir, true);
-        fs.close();
-      }
-      cluster.shutdown();
+      dfs.delete(hdfsTestDir, true);
     }
   }
 
@@ -2496,31 +2335,24 @@ public class TestDFSShell {
   @Test (timeout = 120000)
   public void testCopyCommandsToDirectoryWithPreserveOption()
       throws Exception {
-    Configuration conf = new Configuration();
-    conf.setBoolean(DFSConfigKeys.DFS_NAMENODE_XATTRS_ENABLED_KEY, true);
-    conf.setBoolean(DFSConfigKeys.DFS_NAMENODE_ACLS_ENABLED_KEY, true);
-    MiniDFSCluster cluster = new MiniDFSCluster.Builder(conf).numDataNodes(1)
-        .format(true).build();
     FsShell shell = null;
-    FileSystem fs = null;
     final String testdir =
         "/tmp/TestDFSShell-testCopyCommandsToDirectoryWithPreserveOption-"
         + counter.getAndIncrement();
     final Path hdfsTestDir = new Path(testdir);
     try {
-      fs = cluster.getFileSystem();
-      fs.mkdirs(hdfsTestDir);
+      dfs.mkdirs(hdfsTestDir);
       Path srcDir = new Path(hdfsTestDir, "srcDir");
-      fs.mkdirs(srcDir);
+      dfs.mkdirs(srcDir);
 
-      fs.setAcl(srcDir, Lists.newArrayList(
+      dfs.setAcl(srcDir, Lists.newArrayList(
           aclEntry(ACCESS, USER, ALL),
           aclEntry(ACCESS, USER, "foo", ALL),
           aclEntry(ACCESS, GROUP, READ_EXECUTE),
           aclEntry(DEFAULT, GROUP, "bar", READ_EXECUTE),
           aclEntry(ACCESS, OTHER, EXECUTE)));
       // set sticky bit
-      fs.setPermission(srcDir,
+      dfs.setPermission(srcDir,
           new FsPermission(ALL, READ_EXECUTE, EXECUTE, true));
 
       // Create a file in srcDir to check if modification time of
@@ -2528,19 +2360,19 @@ public class TestDFSShell {
       // If cp -p command is to preserve modification time and then copy child
       // (srcFile), modification time will not be preserved.
       Path srcFile = new Path(srcDir, "srcFile");
-      fs.create(srcFile).close();
+      dfs.create(srcFile).close();
 
-      FileStatus status = fs.getFileStatus(srcDir);
+      FileStatus status = dfs.getFileStatus(srcDir);
       final long mtime = status.getModificationTime();
       final long atime = status.getAccessTime();
       final String owner = status.getOwner();
       final String group = status.getGroup();
       final FsPermission perm = status.getPermission();
 
-      fs.setXAttr(srcDir, USER_A1, USER_A1_VALUE);
-      fs.setXAttr(srcDir, TRUSTED_A1, TRUSTED_A1_VALUE);
+      dfs.setXAttr(srcDir, USER_A1, USER_A1_VALUE);
+      dfs.setXAttr(srcDir, TRUSTED_A1, TRUSTED_A1_VALUE);
 
-      shell = new FsShell(conf);
+      shell = new FsShell(dfs.getConf());
 
       // -p
       Path targetDir1 = new Path(hdfsTestDir, "targetDir1");
@@ -2548,16 +2380,16 @@ public class TestDFSShell {
           targetDir1.toUri().toString() };
       int ret = ToolRunner.run(shell, argv);
       assertEquals("cp -p is not working", SUCCESS, ret);
-      FileStatus targetStatus = fs.getFileStatus(targetDir1);
+      FileStatus targetStatus = dfs.getFileStatus(targetDir1);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       FsPermission targetPerm = targetStatus.getPermission();
       assertTrue(perm.equals(targetPerm));
-      Map<String, byte[]> xattrs = fs.getXAttrs(targetDir1);
+      Map<String, byte[]> xattrs = dfs.getXAttrs(targetDir1);
       assertTrue(xattrs.isEmpty());
-      List<AclEntry> acls = fs.getAclStatus(targetDir1).getEntries();
+      List<AclEntry> acls = dfs.getAclStatus(targetDir1).getEntries();
       assertTrue(acls.isEmpty());
       assertFalse(targetPerm.getAclBit());
 
@@ -2567,16 +2399,16 @@ public class TestDFSShell {
           targetDir2.toUri().toString() };
       ret = ToolRunner.run(shell, argv);
       assertEquals("cp -ptop is not working", SUCCESS, ret);
-      targetStatus = fs.getFileStatus(targetDir2);
+      targetStatus = dfs.getFileStatus(targetDir2);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       targetPerm = targetStatus.getPermission();
       assertTrue(perm.equals(targetPerm));
-      xattrs = fs.getXAttrs(targetDir2);
+      xattrs = dfs.getXAttrs(targetDir2);
       assertTrue(xattrs.isEmpty());
-      acls = fs.getAclStatus(targetDir2).getEntries();
+      acls = dfs.getAclStatus(targetDir2).getEntries();
       assertTrue(acls.isEmpty());
       assertFalse(targetPerm.getAclBit());
 
@@ -2586,18 +2418,18 @@ public class TestDFSShell {
           targetDir3.toUri().toString() };
       ret = ToolRunner.run(shell, argv);
       assertEquals("cp -ptopx is not working", SUCCESS, ret);
-      targetStatus = fs.getFileStatus(targetDir3);
+      targetStatus = dfs.getFileStatus(targetDir3);
       assertEquals(mtime, targetStatus.getModificationTime());
       assertEquals(atime, targetStatus.getAccessTime());
       assertEquals(owner, targetStatus.getOwner());
       assertEquals(group, targetStatus.getGroup());
       targetPerm = targetStatus.getPermissi

<TRUNCATED>

---------------------------------------------------------------------
To unsubscribe, e-mail: common-commits-unsubscribe@hadoop.apache.org
For additional commands, e-mail: common-commits-help@hadoop.apache.org


[2/2] hadoop git commit: HDFS-10893. Refactor TestDFSShell by setting up MiniDFSCluser once for all commands test. Contributed by Mingliang Liu

Posted by li...@apache.org.
HDFS-10893. Refactor TestDFSShell by setting up MiniDFSCluser once for all commands test. Contributed by Mingliang Liu


Project: http://git-wip-us.apache.org/repos/asf/hadoop/repo
Commit: http://git-wip-us.apache.org/repos/asf/hadoop/commit/14bacd2b
Tree: http://git-wip-us.apache.org/repos/asf/hadoop/tree/14bacd2b
Diff: http://git-wip-us.apache.org/repos/asf/hadoop/diff/14bacd2b

Branch: refs/heads/branch-2
Commit: 14bacd2b999c91a3f7a19b38fd63404e6f91c4a0
Parents: 434403a
Author: Mingliang Liu <li...@apache.org>
Authored: Tue Sep 27 14:03:23 2016 -0700
Committer: Mingliang Liu <li...@apache.org>
Committed: Wed Oct 5 22:54:27 2016 -0700

----------------------------------------------------------------------
 .../org/apache/hadoop/hdfs/TestDFSShell.java    | 2107 ++++++++----------
 1 file changed, 908 insertions(+), 1199 deletions(-)
----------------------------------------------------------------------



---------------------------------------------------------------------
To unsubscribe, e-mail: common-commits-unsubscribe@hadoop.apache.org
For additional commands, e-mail: common-commits-help@hadoop.apache.org