فهرست منبع

HDFS-17272. NNThroughputBenchmark should support specifying the base directory for multi-client test (#6319). Contributed by caozhiqiang.

Reviewed-by: Tao Li <tomscut@apache.org>
Signed-off-by: Ayush Saxena <ayushsaxena@apache.org>
caozhiqiang 1 سال پیش
والد
کامیت
37d6cada14

+ 10 - 9
hadoop-common-project/hadoop-common/src/site/markdown/Benchmarking.md

@@ -54,15 +54,15 @@ Following are all the operations supported along with their respective operation
 | OPERATION\_OPTION    | Operation-specific parameters |
 |:---- |:---- |
 |`all` | _options for other operations_ |
-|`create` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-close`] |
-|`mkdirs` | [`-threads 3`] [`-dirs 10`] [`-dirsPerDir 2`] |
-|`open` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] |
-|`delete` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] |
-|`append` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] [`-appendNewBlk`] |
-|`fileStatus` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] |
-|`rename` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] |
-|`blockReport` | [`-datanodes 10`] [`-reports 30`] [`-blocksPerReport 100`] [`-blocksPerFile 10`] |
-|`replication` | [`-datanodes 10`] [`-nodesToDecommission 1`] [`-nodeReplicationLimit 100`] [`-totalBlocks 100`] [`-replication 3`] |
+|`create` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-close`] [`-baseDirName /nnThroughputBenchmark`] |
+|`mkdirs` | [`-threads 3`] [`-dirs 10`] [`-dirsPerDir 2`] [`-baseDirName /nnThroughputBenchmark`] |
+|`open` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] [`-baseDirName /nnThroughputBenchmark`] |
+|`delete` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] [`-baseDirName /nnThroughputBenchmark`] |
+|`append` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] [`-appendNewBlk`] [`-baseDirName /nnThroughputBenchmark`] |
+|`fileStatus` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] [`-baseDirName /nnThroughputBenchmark`] |
+|`rename` | [`-threads 3`] [`-files 10`] [`-filesPerDir 4`] [`-useExisting`] [`-baseDirName /nnThroughputBenchmark`] |
+|`blockReport` | [`-datanodes 10`] [`-reports 30`] [`-blocksPerReport 100`] [`-blocksPerFile 10`] [`-baseDirName /nnThroughputBenchmark`] |
+|`replication` | [`-datanodes 10`] [`-nodesToDecommission 1`] [`-nodeReplicationLimit 100`] [`-totalBlocks 100`] [`-replication 3`] [`-baseDirName /nnThroughputBenchmark`] |
 |`clean` | N/A |
 
 ##### Operation Options
@@ -86,6 +86,7 @@ When running benchmarks with the above operation(s), please provide operation-sp
 |`-nodeReplicationLimit` | The maximum number of outgoing replication streams for a data-node. |
 |`-totalBlocks` | Number of total blocks to operate. |
 |`-replication` | Replication factor. Will be adjusted to number of data-nodes if it is larger than that. |
+|`-baseDirName` | The base dir name for benchmarks, to support multiple clients submitting benchmark tests at the same time. |
 
 ### Reports
 

+ 61 - 24
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/NNThroughputBenchmark.java

@@ -162,11 +162,11 @@ public class NNThroughputBenchmark implements Tool {
    * specific name-node operation.
    */
   abstract class OperationStatsBase {
-    protected static final String BASE_DIR_NAME = "/nnThroughputBenchmark";
+    private String baseDirName = "/nnThroughputBenchmark";
     protected static final String OP_ALL_NAME = "all";
     protected static final String OP_ALL_USAGE = "-op all <other ops options>";
 
-    protected final String baseDir;
+    private String baseDir;
     protected short replication;
     protected int blockSize;
     protected int  numThreads = 0;        // number of threads
@@ -228,7 +228,7 @@ public class NNThroughputBenchmark implements Tool {
     abstract void printResults();
 
     OperationStatsBase() {
-      baseDir = BASE_DIR_NAME + "/" + getOpName();
+      baseDir = baseDirName + "/" + getOpName();
       replication = (short) config.getInt(DFSConfigKeys.DFS_REPLICATION_KEY, 3);
       blockSize = config.getInt(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, BLOCK_SIZE);
       numOpsRequired = 10;
@@ -295,7 +295,12 @@ public class NNThroughputBenchmark implements Tool {
         clientProto.saveNamespace(0, 0);
       }
     }
-
+    public String getBaseDirName() {
+      return baseDirName;
+    }
+    public void setBaseDirName(String baseDirName) {
+      this.baseDirName = baseDirName;
+    }
     int getNumOpsExecuted() {
       return numOpsExecuted;
     }
@@ -316,10 +321,15 @@ public class NNThroughputBenchmark implements Tool {
       return elapsedTime == 0 ? 0 : 1000*(double)numOpsExecuted / elapsedTime;
     }
 
-    String getBaseDir() {
+    public String getBaseDir() {
+      setBaseDir(baseDirName + "/" + getOpName());
       return baseDir;
     }
 
+    public void setBaseDir(String baseDir) {
+      this.baseDir = baseDir;
+    }
+
     String getClientName(int idx) {
       return getOpName() + "-client-" + idx;
     }
@@ -494,7 +504,7 @@ public class NNThroughputBenchmark implements Tool {
       clientProto.setSafeMode(HdfsConstants.SafeModeAction.SAFEMODE_LEAVE,
           false);
       long start = Time.now();
-      clientProto.delete(BASE_DIR_NAME, true);
+      clientProto.delete(getBaseDirName(), true);
       long end = Time.now();
       return end-start;
     }
@@ -502,7 +512,7 @@ public class NNThroughputBenchmark implements Tool {
     @Override
     void printResults() {
       LOG.info("--- " + getOpName() + " inputs ---");
-      LOG.info("Remove directory " + BASE_DIR_NAME);
+      LOG.info("Remove directory " + getBaseDirName());
       printStats();
     }
   }
@@ -517,9 +527,9 @@ public class NNThroughputBenchmark implements Tool {
   class CreateFileStats extends OperationStatsBase {
     // Operation types
     static final String OP_CREATE_NAME = "create";
-    static final String OP_CREATE_USAGE = 
+    static final String OP_CREATE_USAGE =
         "-op create [-threads T] [-files N] [-blockSize S] [-filesPerDir P]"
-        + " [-close]";
+        + " [-baseDirName D] [-close]";
 
     protected FileNameGenerator nameGenerator;
     protected String[][] fileNames;
@@ -553,6 +563,11 @@ public class NNThroughputBenchmark implements Tool {
         } else if(args.get(i).equals("-filesPerDir")) {
           if(i+1 == args.size())  printUsage();
           nrFilesPerDir = Integer.parseInt(args.get(++i));
+        } else if(args.get(i).equals("-baseDirName")) {
+          if (i + 1 == args.size()) {
+            printUsage();
+          }
+          setBaseDirName(args.get(++i));
         } else if(args.get(i).equals("-close")) {
           closeUponCreate = true;
         } else if(!ignoreUnrelatedOptions)
@@ -568,6 +583,7 @@ public class NNThroughputBenchmark implements Tool {
           false);
       // int generatedFileIdx = 0;
       LOG.info("Generate " + numOpsRequired + " intputs for " + getOpName());
+      LOG.info("basedir: " + getBaseDir());
       fileNames = new String[numThreads][];
       try {
         for(int idx=0; idx < numThreads; idx++) {
@@ -618,6 +634,7 @@ public class NNThroughputBenchmark implements Tool {
     @Override
     void printResults() {
       LOG.info("--- " + getOpName() + " inputs ---");
+      LOG.info("baseDir = " + getBaseDir());
       LOG.info("nrFiles = " + numOpsRequired);
       LOG.info("nrThreads = " + numThreads);
       LOG.info("nrFilesPerDir = " + nameGenerator.getFilesPerDirectory());
@@ -635,7 +652,7 @@ public class NNThroughputBenchmark implements Tool {
     // Operation types
     static final String OP_MKDIRS_NAME = "mkdirs";
     static final String OP_MKDIRS_USAGE = "-op mkdirs [-threads T] [-dirs N] " +
-        "[-dirsPerDir P]";
+        "[-dirsPerDir P] [-baseDirName D]";
 
     protected FileNameGenerator nameGenerator;
     protected String[][] dirPaths;
@@ -664,6 +681,11 @@ public class NNThroughputBenchmark implements Tool {
         } else if(args.get(i).equals("-dirsPerDir")) {
           if(i+1 == args.size())  printUsage();
           nrDirsPerDir = Integer.parseInt(args.get(++i));
+        } else if(args.get(i).equals("-baseDirName")) {
+          if (i + 1 == args.size()) {
+            printUsage();
+          }
+          setBaseDirName(args.get(++i));
         } else if(!ignoreUnrelatedOptions)
           printUsage();
       }
@@ -718,6 +740,7 @@ public class NNThroughputBenchmark implements Tool {
     @Override
     void printResults() {
       LOG.info("--- " + getOpName() + " inputs ---");
+      LOG.info("baseDir = " + getBaseDir());
       LOG.info("nrDirs = " + numOpsRequired);
       LOG.info("nrThreads = " + numThreads);
       LOG.info("nrDirsPerDir = " + nameGenerator.getFilesPerDirectory());
@@ -736,7 +759,7 @@ public class NNThroughputBenchmark implements Tool {
     static final String OP_OPEN_NAME = "open";
     static final String OP_USAGE_ARGS = 
         " [-threads T] [-files N] [-blockSize S] [-filesPerDir P]"
-        + " [-useExisting]";
+        + " [-useExisting] [-baseDirName D]";
     static final String OP_OPEN_USAGE = 
       "-op " + OP_OPEN_NAME + OP_USAGE_ARGS;
 
@@ -765,13 +788,14 @@ public class NNThroughputBenchmark implements Tool {
     void generateInputs(int[] opsPerThread) throws IOException {
       // create files using opsPerThread
       String[] createArgs = new String[] {
-              "-op", "create", 
-              "-threads", String.valueOf(this.numThreads), 
-              "-files", String.valueOf(numOpsRequired),
-              "-blockSize", String.valueOf(blockSize),
-              "-filesPerDir", 
-              String.valueOf(nameGenerator.getFilesPerDirectory()),
-              "-close"};
+          "-op", "create",
+          "-threads", String.valueOf(this.numThreads),
+          "-files", String.valueOf(numOpsRequired),
+          "-blockSize", String.valueOf(blockSize),
+          "-filesPerDir",
+          String.valueOf(nameGenerator.getFilesPerDirectory()),
+          "-baseDirName", getBaseDirName(),
+          "-close"};
       CreateFileStats opCreate =  new CreateFileStats(Arrays.asList(createArgs));
 
       if(!useExisting) {  // create files if they were not created before
@@ -1135,9 +1159,9 @@ public class NNThroughputBenchmark implements Tool {
    */
   class BlockReportStats extends OperationStatsBase {
     static final String OP_BLOCK_REPORT_NAME = "blockReport";
-    static final String OP_BLOCK_REPORT_USAGE = 
-      "-op blockReport [-datanodes T] [-reports N] " +
-      "[-blocksPerReport B] [-blocksPerFile F] [-blockSize S]";
+    static final String OP_BLOCK_REPORT_USAGE =
+        "-op blockReport [-datanodes T] [-reports N] " +
+        "[-blocksPerReport B] [-blocksPerFile F] [-blockSize S] [-baseDirName D]";
 
     private int blocksPerReport;
     private int blocksPerFile;
@@ -1187,6 +1211,11 @@ public class NNThroughputBenchmark implements Tool {
         } else if (args.get(i).equals("-blockSize")) {
           if(i+1 == args.size())  printUsage();
           blockSize = Integer.parseInt(args.get(++i));
+        } else if(args.get(i).equals("-baseDirName")) {
+          if (i + 1 == args.size()) {
+            printUsage();
+          }
+          setBaseDirName(args.get(++i));
         } else if(!ignoreUnrelatedOptions)
           printUsage();
       }
@@ -1330,6 +1359,7 @@ public class NNThroughputBenchmark implements Tool {
       }
       blockDistribution += ")";
       LOG.info("--- " + getOpName() + " inputs ---");
+      LOG.info("baseDir = " + getBaseDir());
       LOG.info("reports = " + numOpsRequired);
       LOG.info("datanodes = " + numThreads + " " + blockDistribution);
       LOG.info("blocksPerReport = " + blocksPerReport);
@@ -1348,7 +1378,7 @@ public class NNThroughputBenchmark implements Tool {
     static final String OP_REPLICATION_USAGE = 
         "-op replication [-datanodes T] [-nodesToDecommission D] " +
         "[-nodeReplicationLimit C] [-totalBlocks B] [-blockSize S] "
-        + "[-replication R]";
+        + "[-replication R] [-baseDirName D]";
 
     private final BlockReportStats blockReportObject;
     private int numDatanodes;
@@ -1377,7 +1407,8 @@ public class NNThroughputBenchmark implements Tool {
           "-datanodes", String.valueOf(numDatanodes),
           "-blocksPerReport", String.valueOf(totalBlocks*replication/numDatanodes),
           "-blocksPerFile", String.valueOf(numDatanodes),
-          "-blockSize", String.valueOf(blockSize)};
+          "-blockSize", String.valueOf(blockSize),
+          "-baseDirName", getBaseDirName()};
       blockReportObject = new BlockReportStats(Arrays.asList(blkReportArgs));
       numDecommissionedBlocks = 0;
       numPendingBlocks = 0;
@@ -1410,6 +1441,11 @@ public class NNThroughputBenchmark implements Tool {
         } else if (args.get(i).equals("-blockSize")) {
           if(i+1 == args.size())  printUsage();
           blockSize = Integer.parseInt(args.get(++i));
+        } else if(args.get(i).equals("-baseDirName")) {
+          if (i + 1 == args.size()) {
+            printUsage();
+          }
+          setBaseDirName(args.get(++i));
         } else if(!ignoreUnrelatedOptions)
           printUsage();
       }
@@ -1485,6 +1521,7 @@ public class NNThroughputBenchmark implements Tool {
       }
       blockDistribution += ")";
       LOG.info("--- " + getOpName() + " inputs ---");
+      LOG.info("baseDir = " + getBaseDir());
       LOG.info("numOpsRequired = " + numOpsRequired);
       LOG.info("datanodes = " + numDatanodes + " " + blockDistribution);
       LOG.info("decommissioned datanodes = " + nodesToDecommission);
@@ -1631,7 +1668,7 @@ public class NNThroughputBenchmark implements Tool {
       }
       // run each benchmark
       for(OperationStatsBase op : ops) {
-        LOG.info("Starting benchmark: " + op.getOpName());
+        LOG.info("Starting benchmark: " + op.getOpName() + ", baseDir: " + op.getBaseDir());
         op.benchmark();
         op.cleanUp();
       }

+ 36 - 0
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/namenode/TestNNThroughputBenchmark.java

@@ -22,8 +22,10 @@ import java.io.File;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.FileSystem;
 import org.apache.hadoop.fs.FileUtil;
+import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.hdfs.DFSConfigKeys;
 import org.apache.hadoop.hdfs.DFSTestUtil;
+import org.apache.hadoop.hdfs.DistributedFileSystem;
 import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.hdfs.MiniDFSCluster;
 import org.apache.hadoop.hdfs.protocol.DirectoryListing;
@@ -187,4 +189,38 @@ public class TestNNThroughputBenchmark {
               "blockReport", "-datanodes", "3", "-reports", "2"});
     }
   }
+
+  /**
+   * This test runs {@link NNThroughputBenchmark} against a mini DFS cluster
+   * with explicit -baseDirName option.
+   */
+  @Test(timeout = 120000)
+  public void testNNThroughputWithBaseDir() throws Exception {
+    final Configuration conf = new HdfsConfiguration();
+    conf.setInt(DFSConfigKeys.DFS_NAMENODE_MIN_BLOCK_SIZE_KEY, 16);
+    MiniDFSCluster cluster = null;
+    try {
+      cluster = new MiniDFSCluster.Builder(conf).numDataNodes(0).build();
+      cluster.waitActive();
+      final Configuration benchConf = new HdfsConfiguration();
+      benchConf.setInt(DFSConfigKeys.DFS_BLOCK_SIZE_KEY, 16);
+      FileSystem.setDefaultUri(benchConf, cluster.getURI());
+      DistributedFileSystem fs = cluster.getFileSystem();
+
+      NNThroughputBenchmark.runBenchmark(benchConf,
+          new String[] {"-op", "create", "-keepResults", "-files", "3", "-baseDirName",
+              "/nnThroughputBenchmark1", "-close"});
+      Assert.assertTrue(fs.exists(new Path("/nnThroughputBenchmark1")));
+      Assert.assertFalse(fs.exists(new Path("/nnThroughputBenchmark")));
+
+      NNThroughputBenchmark.runBenchmark(benchConf,
+          new String[] {"-op", "all", "-baseDirName", "/nnThroughputBenchmark1"});
+      Assert.assertTrue(fs.exists(new Path("/nnThroughputBenchmark1")));
+      Assert.assertFalse(fs.exists(new Path("/nnThroughputBenchmark")));
+    } finally {
+      if (cluster != null) {
+        cluster.shutdown();
+      }
+    }
+  }
 }