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 om...@apache.org on 2008/10/21 08:27:22 UTC

svn commit: r706537 [3/3] - in /hadoop/core/branches/branch-0.19: ./ conf/ docs/ src/docs/src/documentation/content/xdocs/ src/mapred/org/apache/hadoop/mapred/ src/test/org/apache/hadoop/mapred/

Modified: hadoop/core/branches/branch-0.19/src/docs/src/documentation/content/xdocs/mapred_tutorial.xml
URL: http://svn.apache.org/viewvc/hadoop/core/branches/branch-0.19/src/docs/src/documentation/content/xdocs/mapred_tutorial.xml?rev=706537&r1=706536&r2=706537&view=diff
==============================================================================
--- hadoop/core/branches/branch-0.19/src/docs/src/documentation/content/xdocs/mapred_tutorial.xml (original)
+++ hadoop/core/branches/branch-0.19/src/docs/src/documentation/content/xdocs/mapred_tutorial.xml Mon Oct 20 23:27:22 2008
@@ -1097,27 +1097,6 @@
         <a href="cluster_setup.html#Configuring+the+Environment+of+the+Hadoop+Daemons">
         cluster_setup.html </a></p>
         
-        <p>There are two additional parameters that influence virtual memory
-        limits for tasks run on a tasktracker. The parameter 
-        <code>mapred.tasktracker.maxmemory</code> is set by admins
-        to limit the total memory all tasks that it runs can use together. 
-        Setting this enables the parameter <code>mapred.task.maxmemory</code>
-        that can be used to specify the maximum virtual memory the entire 
-        process tree starting from the launched child-task requires. 
-        This is a cumulative limit of all processes in the process tree. 
-        By specifying this value, users can be assured that the system will 
-        run their tasks only on tasktrackers that have atleast this amount 
-        of free memory available. If at any time during task execution, this 
-        limit is exceeded, the task would be killed by the system. By default, 
-        any task would get a share of 
-        <code>mapred.tasktracker.maxmemory</code>, divided
-        equally among the number of slots. The user can thus verify if the
-        tasks need more memory than this, and specify it in 
-        <code>mapred.task.maxmemory</code>. Specifically, this value must be 
-        greater than any value specified for a maximum heap-size
-        of the child jvm via <code>mapred.child.java.opts</code>, or a ulimit
-        value in <code>mapred.child.ulimit</code>. </p>
-
         <p>The memory available to some parts of the framework is also
         configurable. In map and reduce tasks, performance may be influenced
         by adjusting parameters influencing the concurrency of operations and

Modified: hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/JobConf.java
URL: http://svn.apache.org/viewvc/hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/JobConf.java?rev=706537&r1=706536&r2=706537&view=diff
==============================================================================
--- hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/JobConf.java (original)
+++ hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/JobConf.java Mon Oct 20 23:27:22 2008
@@ -108,7 +108,7 @@
    * A value which if set for memory related configuration options,
    * indicates that the options are turned off.
    */
-  public static final long DISABLED_VIRTUAL_MEMORY_LIMIT = -1L;
+  static final long DISABLED_VIRTUAL_MEMORY_LIMIT = -1L;
   
   /**
    * Name of the queue to which jobs will be submitted, if no queue
@@ -1337,37 +1337,6 @@
   }
   
   /**
-   * The maximum amount of virtual memory all tasks running on a
-   * tasktracker, including sub-processes they launch, can use.
-   *  
-   * This value is used to compute the amount of free memory 
-   * available for tasks. Any task scheduled on this tasktracker is 
-   * guaranteed and constrained to use a share of this amount. Any task 
-   * exceeding its share will be killed.
-   * 
-   * If set to {@link #DISABLED_VIRTUAL_MEMORY_LIMIT}, this functionality 
-   * is disabled.
-   * 
-   * @return maximum amount of virtual memory in kilobytes to divide among
-   * @see #getMaxVirtualMemoryForTask()
-   */
-  public long getMaxVirtualMemoryForTasks() {
-    return getLong("mapred.tasktracker.tasks.maxmemory", 
-                      DISABLED_VIRTUAL_MEMORY_LIMIT);
-  }
-  
-  /**
-   * Set the maximum amount of virtual memory all tasks running on a
-   * tasktracker, including sub-processes they launch, can use.
-   * 
-   * @param vmem maximum amount of virtual memory in kilobytes that can be used.
-   * @see #getMaxVirtualMemoryForTasks()
-   */
-  public void setMaxVirtualMemoryForTasks(long vmem) {
-    setLong("mapred.tasktracker.tasks.maxmemory", vmem);
-  }
-  
-  /**
    * The maximum amount of memory any task of this job will use.
    * 
    * A task of this job will be scheduled on a tasktracker, only if the
@@ -1375,15 +1344,14 @@
    * or equal to this value.
    * 
    * If set to {@link #DISABLED_VIRTUAL_MEMORY_LIMIT}, tasks are assured 
-   * a memory limit on the tasktracker equal to
-   * mapred.tasktracker.tasks.maxmemory/number of slots. If the value of
+   * a memory limit set to mapred.task.default.maxmemory. If the value of
    * mapred.tasktracker.tasks.maxmemory is set to -1, this value is 
    * ignored.
    * 
    * @return The maximum amount of memory any task of this job will use, in kilobytes.
    * @see #getMaxVirtualMemoryForTasks()
    */
-  public long getMaxVirtualMemoryForTask() {
+  long getMaxVirtualMemoryForTask() {
     return getLong("mapred.task.maxmemory", DISABLED_VIRTUAL_MEMORY_LIMIT);
   }
   
@@ -1394,7 +1362,7 @@
    * can use.
    * @see #getMaxVirtualMemoryForTask()
    */
-  public void setMaxVirtualMemoryForTask(long vmem) {
+  void setMaxVirtualMemoryForTask(long vmem) {
     setLong("mapred.task.maxmemory", vmem);
   }
   

Modified: hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTracker.java
URL: http://svn.apache.org/viewvc/hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTracker.java?rev=706537&r1=706536&r2=706537&view=diff
==============================================================================
--- hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTracker.java (original)
+++ hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTracker.java Mon Oct 20 23:27:22 2008
@@ -197,8 +197,6 @@
   private boolean taskMemoryManagerEnabled = false;
   private long maxVirtualMemoryForTasks 
                                     = JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT;
-  private long defaultMemoryPerTask = JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT;
-  
   
   /**
    * the minimum interval between jobtracker polls
@@ -471,13 +469,9 @@
                              "Map-events fetcher for all reduce tasks " + "on " + 
                              taskTrackerName);
     mapEventsFetcher.start();
-    maxVirtualMemoryForTasks = fConf.getMaxVirtualMemoryForTasks();
-    if (maxVirtualMemoryForTasks != 
-                JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT) {
-      defaultMemoryPerTask = maxVirtualMemoryForTasks /
-                                    (maxCurrentMapTasks + 
-                                        maxCurrentReduceTasks);
-    }
+    maxVirtualMemoryForTasks = fConf.
+                                  getLong("mapred.tasktracker.tasks.maxmemory",
+                                          JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT);
     this.indexCache = new IndexCache(this.fConf);
     // start the taskMemoryManager thread only if enabled
     setTaskMemoryManagerEnabledFlag();
@@ -785,10 +779,6 @@
     launchTaskForJob(tip, new JobConf(rjob.jobFile)); 
   }
 
-  private long getDefaultMemoryPerTask() {
-    return defaultMemoryPerTask;
-  }
-
   private void launchTaskForJob(TaskInProgress tip, JobConf jobConf) throws IOException{
     synchronized (tip) {
       try {
@@ -1180,7 +1170,7 @@
       LOG.debug("Setting amount of free virtual memory for the new task: " +
                     freeVirtualMem);
       status.getResourceStatus().setFreeVirtualMemory(freeVirtualMem);
-      status.getResourceStatus().setDefaultVirtualMemoryPerTask(getDefaultMemoryPerTask());      
+      status.getResourceStatus().setTotalMemory(maxVirtualMemoryForTasks);
     }
       
     //
@@ -1279,10 +1269,11 @@
    * @param conf
    * @return the memory allocated for the TIP.
    */
-  public long getMemoryForTask(JobConf conf) {
+  long getMemoryForTask(JobConf conf) {
     long memForTask = conf.getMaxVirtualMemoryForTask();
     if (memForTask == JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT) {
-      memForTask = this.getDefaultMemoryPerTask();
+      memForTask = fConf.getLong("mapred.task.default.maxmemory",
+                          512*1024*1024L);
     }
     return memForTask;
   }  

Modified: hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTrackerStatus.java
URL: http://svn.apache.org/viewvc/hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTrackerStatus.java?rev=706537&r1=706536&r2=706537&view=diff
==============================================================================
--- hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTrackerStatus.java (original)
+++ hadoop/core/branches/branch-0.19/src/mapred/org/apache/hadoop/mapred/TaskTrackerStatus.java Mon Oct 20 23:27:22 2008
@@ -55,12 +55,12 @@
   static class ResourceStatus implements Writable {
     
     private long freeVirtualMemory;
-    private long defaultVirtualMemoryPerTask;
+    private long totalMemory;
     private long availableSpace;
     
     ResourceStatus() {
       freeVirtualMemory = JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT;
-      defaultVirtualMemoryPerTask = JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT;
+      totalMemory = JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT;
       availableSpace = Long.MAX_VALUE;
     }
     
@@ -87,25 +87,24 @@
     }
 
     /**
-     * Set the default amount of virtual memory per task.
-     * @param vmem amount of free virtual memory in kilobytes.
+     * Set the maximum amount of virtual memory on the tasktracker.
+     * @param vmem maximum amount of virtual memory on the tasktracker in kilobytes.
      */
-    void setDefaultVirtualMemoryPerTask(long defaultVmem) {
-      defaultVirtualMemoryPerTask = defaultVmem;
+    void setTotalMemory(long totalMem) {
+      totalMemory = totalMem;
     }
     
     /**
-     * Get the default amount of virtual memory per task.
+     * Get the maximum amount of virtual memory on the tasktracker.
      * 
-     * This amount will be returned if a task's job does not specify any
-     * virtual memory itself. If this is 
+     * If this is
      * {@link JobConf.DISABLED_VIRTUAL_MEMORY_LIMIT}, it should be ignored 
      * and not used in any computation.
      * 
-     * @return default amount of virtual memory per task in kilobytes. 
+     * @return maximum amount of virtual memory on the tasktracker in kilobytes. 
      */    
-    long getDefaultVirtualMemoryPerTask() {
-      return defaultVirtualMemoryPerTask;
+    long getTotalMemory() {
+      return totalMemory;
     }
     
     void setAvailableSpace(long availSpace) {
@@ -122,13 +121,13 @@
     
     public void write(DataOutput out) throws IOException {
       WritableUtils.writeVLong(out, freeVirtualMemory);
-      WritableUtils.writeVLong(out, defaultVirtualMemoryPerTask);
+      WritableUtils.writeVLong(out, totalMemory);
       WritableUtils.writeVLong(out, availableSpace);
     }
     
     public void readFields(DataInput in) throws IOException {
       freeVirtualMemory = WritableUtils.readVLong(in);;
-      defaultVirtualMemoryPerTask = WritableUtils.readVLong(in);;
+      totalMemory = WritableUtils.readVLong(in);;
       availableSpace = WritableUtils.readVLong(in);;
     }
   }

Modified: hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestHighRAMJobs.java
URL: http://svn.apache.org/viewvc/hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestHighRAMJobs.java?rev=706537&r1=706536&r2=706537&view=diff
==============================================================================
--- hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestHighRAMJobs.java (original)
+++ hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestHighRAMJobs.java Mon Oct 20 23:27:22 2008
@@ -78,7 +78,7 @@
       TestHighRAMJobs.LOG.info("status = " + status.getResourceStatus().getFreeVirtualMemory());
 
       long initialFreeMemory = getConf().getLong("initialFreeMemory", 0L);
-      long memoryPerTaskOnTT = getConf().getLong("memoryPerTaskOnTT", 0L);
+      long totalMemoryOnTT = getConf().getLong("totalMemoryOnTT", 0L);
 
       if (isFirstTime) {
         isFirstTime = false;
@@ -87,19 +87,15 @@
           message = "Initial memory expected = " + initialFreeMemory
                       + " reported = " + status.getResourceStatus().getFreeVirtualMemory();
         }
-        if (memoryPerTaskOnTT != status.getResourceStatus().getDefaultVirtualMemoryPerTask()) {
+        if (totalMemoryOnTT != status.getResourceStatus().getTotalMemory()) {
           hasPassed = false;
-          message = "Memory per task on TT expected = " + memoryPerTaskOnTT
+          message = "Total memory on TT expected = " + totalMemoryOnTT
                       + " reported = " 
-                      + status.getResourceStatus().getDefaultVirtualMemoryPerTask();
+                      + status.getResourceStatus().getTotalMemory();
         }
       } else if (initialFreeMemory != DISABLED_VIRTUAL_MEMORY_LIMIT) {
         
-        long memoryPerTask = memoryPerTaskOnTT; // by default
-        if (getConf().getLong("memoryPerTask", 0L) != 
-                                            DISABLED_VIRTUAL_MEMORY_LIMIT) {
-          memoryPerTask = getConf().getLong("memoryPerTask", 0L);
-        }
+        long memoryPerTask = getConf().getLong("memoryPerTask", 0L);
           
         long expectedFreeMemory = 0;
         int runningTaskCount = status.countMapTasks() +
@@ -127,8 +123,7 @@
   public void testDefaultValuesForHighRAMJobs() throws Exception {
     long defaultMemoryLimit = DISABLED_VIRTUAL_MEMORY_LIMIT;
     try {
-      setUpCluster(defaultMemoryLimit, defaultMemoryLimit, 
-                    defaultMemoryLimit, null);
+      setUpCluster(defaultMemoryLimit, defaultMemoryLimit, null);
       runJob(defaultMemoryLimit, DEFAULT_MAP_SLEEP_TIME, 
           DEFAULT_REDUCE_SLEEP_TIME, DEFAULT_SLEEP_JOB_MAP_COUNT, 
           DEFAULT_SLEEP_JOB_REDUCE_COUNT);
@@ -142,35 +137,15 @@
    * when the number of slots is non-default.
    */
   public void testDefaultMemoryPerTask() throws Exception {
-    long maxVmem = 1024*1024*1024L;
+    long maxVmem = 2*1024*1024*1024L;
     JobConf conf = new JobConf();
-    conf.setInt("mapred.tasktracker.map.tasks.maximum", 1);
-    conf.setInt("mapred.tasktracker.reduce.tasks.maximum", 1);
-    // change number of slots to 2.
-    long defaultMemPerTaskOnTT = maxVmem / 2;
+    conf.setInt("mapred.tasktracker.map.tasks.maximum", 2);
+    conf.setInt("mapred.tasktracker.reduce.tasks.maximum", 2);
+    // set a different value for the default memory per task
+    long defaultMemPerTask = 256*1024*1024L; 
     try {
-      setUpCluster(maxVmem, defaultMemPerTaskOnTT, 
-                    DISABLED_VIRTUAL_MEMORY_LIMIT, conf);
-      runJob(DISABLED_VIRTUAL_MEMORY_LIMIT, DEFAULT_MAP_SLEEP_TIME,
-              DEFAULT_REDUCE_SLEEP_TIME, DEFAULT_SLEEP_JOB_MAP_COUNT,
-              DEFAULT_SLEEP_JOB_REDUCE_COUNT);
-      verifyTestResults();
-    } finally {
-      tearDownCluster();
-    }
-  }
-  
-  /* Test that verifies configured value for free memory is
-   * reported correctly. The test does NOT configure a value for
-   * memory per task. Hence, it also verifies that the default value
-   * per task on the TT is calculated correctly.
-   */
-  public void testConfiguredValueForFreeMemory() throws Exception {
-    long maxVmem = 1024*1024*1024L;
-    long defaultMemPerTaskOnTT = maxVmem/4; // 4 = default number of slots.
-    try {
-      setUpCluster(maxVmem, defaultMemPerTaskOnTT,
-                    DISABLED_VIRTUAL_MEMORY_LIMIT, null);
+      setUpCluster(maxVmem, defaultMemPerTask, 
+                    defaultMemPerTask, conf);
       runJob(DISABLED_VIRTUAL_MEMORY_LIMIT, "10000",
               DEFAULT_REDUCE_SLEEP_TIME, DEFAULT_SLEEP_JOB_MAP_COUNT,
               DEFAULT_SLEEP_JOB_REDUCE_COUNT);
@@ -182,15 +157,14 @@
   
   public void testHighRAMJob() throws Exception {
     long maxVmem = 1024*1024*1024L;
-    long defaultMemPerTaskOnTT = maxVmem/4; // 4 = default number of slots.
+    //long defaultMemPerTaskOnTT = maxVmem/4; // 4 = default number of slots.
     /* Set a HIGH RAM requirement for a job. As 4 is the
      * default number of slots, we set up the memory limit
      * per task to be more than 25%. 
      */
     long maxVmemPerTask = maxVmem/3;
     try {
-      setUpCluster(maxVmem, defaultMemPerTaskOnTT,
-                    maxVmemPerTask, null);
+      setUpCluster(maxVmem, maxVmemPerTask, null);
       /* set up sleep limits higher, so the scheduler will see varying
        * number of running tasks at a time. Also modify the number of
        * map tasks so we test the iteration over more than one task.
@@ -203,20 +177,27 @@
     }
   }
   
-  private void setUpCluster(long initialFreeMemory, long memoryPerTaskOnTT,
-                            long memoryPerTask, JobConf conf) 
-                              throws Exception {
+  private void setUpCluster(long totalMemoryOnTT, long memoryPerTask,
+                              JobConf conf) throws Exception {
+    this.setUpCluster(totalMemoryOnTT, 512*1024*1024L, 
+                          memoryPerTask, conf);
+  }
+  
+  private void setUpCluster(long totalMemoryOnTT, long defaultMemoryPerTask,
+                              long memoryPerTask, JobConf conf)
+                                throws Exception {
     if (conf == null) {
       conf = new JobConf();
     }
     conf.setClass("mapred.jobtracker.taskScheduler", 
         TestHighRAMJobs.FakeTaskScheduler.class,
         TaskScheduler.class);
-    if (initialFreeMemory != -1L) {
-      conf.setMaxVirtualMemoryForTasks(initialFreeMemory);  
+    if (totalMemoryOnTT != -1L) {
+      conf.setLong("mapred.tasktracker.tasks.maxmemory", totalMemoryOnTT);  
     }
-    conf.setLong("initialFreeMemory", initialFreeMemory);
-    conf.setLong("memoryPerTaskOnTT", memoryPerTaskOnTT);
+    conf.setLong("mapred.task.default.maxmemory", defaultMemoryPerTask);
+    conf.setLong("initialFreeMemory", totalMemoryOnTT);
+    conf.setLong("totalMemoryOnTT", totalMemoryOnTT);
     conf.setLong("memoryPerTask", memoryPerTask);
     miniDFSCluster = new MiniDFSCluster(conf, 1, true, null);
     FileSystem fileSys = miniDFSCluster.getFileSystem();

Modified: hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestTaskTrackerMemoryManager.java
URL: http://svn.apache.org/viewvc/hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestTaskTrackerMemoryManager.java?rev=706537&r1=706536&r2=706537&view=diff
==============================================================================
--- hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestTaskTrackerMemoryManager.java (original)
+++ hadoop/core/branches/branch-0.19/src/test/org/apache/hadoop/mapred/TestTaskTrackerMemoryManager.java Mon Oct 20 23:27:22 2008
@@ -104,7 +104,8 @@
     // Start cluster with proper configuration.
     JobConf fConf = new JobConf();
 
-    fConf.setMaxVirtualMemoryForTasks(Long.valueOf(10000000000L)); // Fairly large value for WordCount to succeed
+    fConf.setLong("mapred.tasktracker.tasks.maxmemory", 
+                      Long.valueOf(10000000000L)); // Fairly large value for WordCount to succeed
     startCluster(fConf);
 
     // Set up job.
@@ -178,7 +179,7 @@
 
     // Start cluster with proper configuration.
     JobConf fConf = new JobConf();
-    fConf.setMaxVirtualMemoryForTasks(Long.valueOf(100000));
+    fConf.setLong("mapred.tasktracker.tasks.maxmemory", Long.valueOf(100000));
     fConf.set("mapred.tasktracker.taskmemorymanager.monitoring-interval", String.valueOf(300));
             //very small value, so that no task escapes to successful completion.
     startCluster(fConf);