Переглянути джерело

commit c263f9b140cf4caac99a56eda2b4dc27b944dad7
Author: Hong Tang <htang@yahoo-inc.com>
Date: Thu Jul 22 16:26:29 2010 -0700

MAPREDUCE-1936. Make Gridmix3 more customizable (sync changes from trunk). From https://issues.apache.org/jira/secure/attachment/12450235/mr-1936-delta-20.1xx.patch. (htang)

+++ b/YAHOO-CHANGES.txt
+ MAPREDUCE-1936. Make Gridmix3 more customizable (sync changes from trunk). (htang)
+


git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-0.20-security-patches@1077580 13f79535-47bb-0310-9956-ffa450edef68

Owen O'Malley 14 роки тому
батько
коміт
6c3d00cf1a

+ 0 - 63
src/contrib/gridmix/src/java/org/apache/hadoop/mapred/gridmix/FilePool.java

@@ -137,69 +137,6 @@ class FilePool {
         throws IOException;
   }
 
-  interface IndexMapper {
-    int get(int pos);
-    void swap(int a, int b);
-  }
-
-  /**
-   * A sparse index mapping table - useful when we want to
-   * non-destructively permute a small fraction of a large array.
-   */
-  static class SparseIndexMapper implements IndexMapper {
-    Map<Integer, Integer> mapping = new HashMap<Integer, Integer>();
-
-    public int get(int pos) {
-      Integer mapped = mapping.get(pos);
-      if (mapped == null) return pos;
-      return mapped;
-    }
-
-    public void swap(int a, int b) {
-      int valA = get(a);
-      int valB = get(b);
-      if (b == valA) {
-        mapping.remove(b);
-      } else {
-        mapping.put(b, valA);
-      }
-      if (a == valB) {
-        mapping.remove(a);
-      } else {
-        mapping.put(a, valB);
-      }
-    }
-  }
-
-  /**
-   * A dense index mapping table - useful when we want to
-   * non-destructively permute a large fraction of an array.
-   */
-  static class DenseIndexMapper implements IndexMapper {
-    int[] mapping;
-
-    DenseIndexMapper(int size) {
-      mapping = new int[size];
-      for (int i=0; i<size; ++i) {
-        mapping[i] = i;
-      }
-    }
-
-    public int get(int pos) {
-      if ( (pos < 0) || (pos>=mapping.length) ) {
-        throw new IndexOutOfBoundsException();
-      }
-      return mapping[pos];
-    }
-
-    public void swap(int a, int b) {
-      int valA = get(a);
-      int valB = get(b);
-      mapping[a]=valB;
-      mapping[b]=valA;
-    }
-  }
-
   /**
    * Files in current directory of this Node.
    */

+ 44 - 30
src/contrib/gridmix/src/java/org/apache/hadoop/mapred/gridmix/Gridmix.java

@@ -394,52 +394,66 @@ public class Gridmix extends Configured implements Tool {
     }
   }
 
+  private <T> String getEnumValues(Enum<? extends T>[] e) {
+    StringBuilder sb = new StringBuilder();
+    String sep = "";
+    for (Enum<? extends T> v : e) {
+      sb.append(sep);
+      sb.append(v.name());
+      sep = "|";
+    }
+    return sb.toString();
+  }
+  
+  private String getJobTypes() {
+    return getEnumValues(JobCreator.values());
+  }
+  
+  private String getSubmissionPolicies() {
+    return getEnumValues(GridmixJobSubmissionPolicy.values());
+  }
+  
   protected void printUsage(PrintStream out) {
     ToolRunner.printGenericCommandUsage(out);
-    out.println("Usage: gridmix [-generate <MiB>] [-users URI] <iopath> <trace>");
+    out.println("Usage: gridmix [-generate <MiB>] [-users URI] [-Dname=value ...] <iopath> <trace>");
     out.println("  e.g. gridmix -generate 100m foo -");
     out.println("Configuration parameters:");
+    out.println("   General parameters:");
     out.printf("       %-48s : Output directory\n", GRIDMIX_OUT_DIR);
     out.printf("       %-48s : Submitting threads\n", GRIDMIX_SUB_THR);
     out.printf("       %-48s : Queued job desc\n", GRIDMIX_QUE_DEP);
-    out.printf("       %-48s : Key fraction of rec\n",
-        AvgRecordFactory.GRIDMIX_KEY_FRC);
     out.printf("       %-48s : User resolution class\n", GRIDMIX_USR_RSV);
-    out.printf("       %-48s : Enable/disable using queues in trace\n",
-        GridmixJob.GRIDMIX_USE_QUEUE_IN_TRACE);
+    out.printf("       %-48s : Job types (%s)\n", JobCreator.GRIDMIX_JOB_TYPE, getJobTypes());
+    out.println("   Parameters related to job submission:");    
     out.printf("       %-48s : Default queue\n",
         GridmixJob.GRIDMIX_DEFAULT_QUEUE);
-    out.printf("       %-48s : Throttling - jobs vs task-tracker ratio\n",
-        StressJobFactory.CONF_MAX_JOB_TRACKER_RATIO);
-    out.printf("       %-48s : Throttling - maps vs map-slot ratio\n",
-        StressJobFactory.CONF_OVERLOAD_MAPTASK_MAPSLOT_RATIO);
-    out.printf("       %-48s : Throttling - reduces vs reduce-slot ratio\n",
-        StressJobFactory.CONF_OVERLOAD_REDUCETASK_REDUCESLOT_RATIO);
-    out.printf("       %-48s : Throttling - map-slot share per job\n",
-        StressJobFactory.CONF_MAX_MAPSLOT_SHARE_PER_JOB);
-    out.printf("       %-48s : Throttling - reduce-slot share per job\n",
-        StressJobFactory.CONF_MAX_REDUCESLOT_SHARE_PER_JOB);
-    out.printf("       %-48s : Whether to ignore reduce tasks for sleep jobs\n",
+    out.printf("       %-48s : Enable/disable using queues in trace\n",
+        GridmixJob.GRIDMIX_USE_QUEUE_IN_TRACE);
+    out.printf("       %-48s : Job submission policy (%s)\n",
+        GridmixJobSubmissionPolicy.JOB_SUBMISSION_POLICY, getSubmissionPolicies());
+    out.println("   Parameters specific for LOADJOB:");
+    out.printf("       %-48s : Key fraction of rec\n",
+        AvgRecordFactory.GRIDMIX_KEY_FRC);
+    out.println("   Parameters specific for SLEEPJOB:");
+    out.printf("       %-48s : Whether to ignore reduce tasks\n",
         SleepJob.SLEEPJOB_MAPTASK_ONLY);
-    out.printf("       %-48s : Number of fake locations for sleep job map tasks\n",
+    out.printf("       %-48s : Number of fake locations for map tasks\n",
         JobCreator.SLEEPJOB_RANDOM_LOCATIONS);
-    
-    
-    StringBuilder sb = new StringBuilder();
-    String sep = "";
-    for (GridmixJobSubmissionPolicy policy : GridmixJobSubmissionPolicy
-        .values()) {
-      sb.append(sep);
-      sb.append(policy.name());
-      sep = "|";
-    }
-    out.printf("       %-48s : Job submission policy (%s)\n",
-        GridmixJobSubmissionPolicy.JOB_SUBMISSION_POLICY, sb.toString());
     out.printf("       %-48s : Maximum map task runtime in mili-sec\n",
         SleepJob.GRIDMIX_SLEEP_MAX_MAP_TIME);
     out.printf("       %-48s : Maximum reduce task runtime in mili-sec (merge+reduce)\n",
         SleepJob.GRIDMIX_SLEEP_MAX_REDUCE_TIME);
-
+    out.println("   Parameters specific for STRESS submission throttling policy:");
+    out.printf("       %-48s : jobs vs task-tracker ratio\n",
+        StressJobFactory.CONF_MAX_JOB_TRACKER_RATIO);
+    out.printf("       %-48s : maps vs map-slot ratio\n",
+        StressJobFactory.CONF_OVERLOAD_MAPTASK_MAPSLOT_RATIO);
+    out.printf("       %-48s : reduces vs reduce-slot ratio\n",
+        StressJobFactory.CONF_OVERLOAD_REDUCETASK_REDUCESLOT_RATIO);
+    out.printf("       %-48s : map-slot share per job\n",
+        StressJobFactory.CONF_MAX_MAPSLOT_SHARE_PER_JOB);
+    out.printf("       %-48s : reduce-slot share per job\n",
+        StressJobFactory.CONF_MAX_REDUCESLOT_SHARE_PER_JOB);
   }
 
   /**

+ 1 - 1
src/contrib/gridmix/src/java/org/apache/hadoop/mapred/gridmix/SleepJob.java

@@ -77,7 +77,7 @@ public class SleepJob extends GridmixJob {
   public static final String GRIDMIX_SLEEP_MAX_REDUCE_TIME = 
     "gridmix.sleep.max-reduce-time";
 
-  private long mapMaxSleepTime, reduceMaxSleepTime;
+  private final long mapMaxSleepTime, reduceMaxSleepTime;
 
   public SleepJob(Configuration conf, long submissionMillis, JobStory jobdesc,
       Path outRoot, UserGroupInformation ugi, int seq, int numLocations,