Przeglądaj źródła

svn merge -c 1327304 from trunk. FIXES: MAPREDUCE-4134. Remove references of mapred.child.ulimit etc. since they are not being used any more (Ravi Prakash via bobby)

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-0.23@1327308 13f79535-47bb-0310-9956-ffa450edef68
Robert Joseph Evans 13 lat temu
rodzic
commit
924c8e703d

+ 3 - 0
hadoop-mapreduce-project/CHANGES.txt

@@ -21,6 +21,9 @@ Release 0.23.3 - UNRELEASED
     MAPREDUCE-4151. RM scheduler web page should filter apps to those that 
     MAPREDUCE-4151. RM scheduler web page should filter apps to those that 
     are relevant to scheduling (Jason Lowe via tgraves)
     are relevant to scheduling (Jason Lowe via tgraves)
 
 
+    MAPREDUCE-4134. Remove references of mapred.child.ulimit etc. since they
+    are not being used any more (Ravi Prakash via bobby)
+
   OPTIMIZATIONS
   OPTIMIZATIONS
 
 
   BUG FIXES
   BUG FIXES

+ 26 - 30
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapred/JobConf.java

@@ -187,9 +187,6 @@ public class JobConf extends Configuration {
    * /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
    * /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
    *          -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
    *          -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
    * 
    * 
-   * The configuration variable {@link #MAPRED_TASK_ULIMIT} can be used to 
-   * control the maximum virtual memory of the child processes.
-   * 
    * The configuration variable {@link #MAPRED_TASK_ENV} can be used to pass 
    * The configuration variable {@link #MAPRED_TASK_ENV} can be used to pass 
    * other environment variables to the child processes.
    * other environment variables to the child processes.
    * 
    * 
@@ -210,9 +207,6 @@ public class JobConf extends Configuration {
    * /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
    * /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
    *          -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
    *          -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
    * 
    * 
-   * The configuration variable {@link #MAPRED_MAP_TASK_ULIMIT} can be used to 
-   * control the maximum virtual memory of the map processes.
-   * 
    * The configuration variable {@link #MAPRED_MAP_TASK_ENV} can be used to pass 
    * The configuration variable {@link #MAPRED_MAP_TASK_ENV} can be used to pass 
    * other environment variables to the map processes.
    * other environment variables to the map processes.
    */
    */
@@ -230,9 +224,6 @@ public class JobConf extends Configuration {
    * /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
    * /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
    *          -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
    *          -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
    * 
    * 
-   * The configuration variable {@link #MAPRED_REDUCE_TASK_ULIMIT} can be used  
-   * to control the maximum virtual memory of the reduce processes.
-   * 
    * The configuration variable {@link #MAPRED_REDUCE_TASK_ENV} can be used to 
    * The configuration variable {@link #MAPRED_REDUCE_TASK_ENV} can be used to 
    * pass process environment variables to the reduce processes.
    * pass process environment variables to the reduce processes.
    */
    */
@@ -242,36 +233,32 @@ public class JobConf extends Configuration {
   public static final String DEFAULT_MAPRED_TASK_JAVA_OPTS = "-Xmx200m";
   public static final String DEFAULT_MAPRED_TASK_JAVA_OPTS = "-Xmx200m";
   
   
   /**
   /**
-   * Configuration key to set the maximum virutal memory available to the child
-   * map and reduce tasks (in kilo-bytes).
-   * 
-   * Note: This must be greater than or equal to the -Xmx passed to the JavaVM
-   *       via {@link #MAPRED_TASK_JAVA_OPTS}, else the VM might not start.
-   * 
-   * @deprecated Use {@link #MAPRED_MAP_TASK_ULIMIT} or 
-   *                 {@link #MAPRED_REDUCE_TASK_ULIMIT}
+   * @deprecated
+   * Configuration key to set the maximum virtual memory available to the child
+   * map and reduce tasks (in kilo-bytes). This has been deprecated and will no
+   * longer have any effect.
    */
    */
   @Deprecated
   @Deprecated
   public static final String MAPRED_TASK_ULIMIT = "mapred.child.ulimit";
   public static final String MAPRED_TASK_ULIMIT = "mapred.child.ulimit";
 
 
   /**
   /**
-   * Configuration key to set the maximum virutal memory available to the
-   * map tasks (in kilo-bytes).
-   * 
-   * Note: This must be greater than or equal to the -Xmx passed to the JavaVM
-   *       via {@link #MAPRED_MAP_TASK_JAVA_OPTS}, else the VM might not start.
+   * @deprecated
+   * Configuration key to set the maximum virtual memory available to the
+   * map tasks (in kilo-bytes). This has been deprecated and will no
+   * longer have any effect.
    */
    */
-  public static final String MAPRED_MAP_TASK_ULIMIT = JobContext.MAP_ULIMIT;
+  @Deprecated
+  public static final String MAPRED_MAP_TASK_ULIMIT = "mapreduce.map.ulimit";
   
   
   /**
   /**
-   * Configuration key to set the maximum virutal memory available to the
-   * reduce tasks (in kilo-bytes).
-   * 
-   * Note: This must be greater than or equal to the -Xmx passed to the JavaVM
-   *       via {@link #MAPRED_REDUCE_TASK_JAVA_OPTS}, else the VM might not start.
+   * @deprecated
+   * Configuration key to set the maximum virtual memory available to the
+   * reduce tasks (in kilo-bytes). This has been deprecated and will no
+   * longer have any effect.
    */
    */
-  public static final String MAPRED_REDUCE_TASK_ULIMIT = 
-    JobContext.REDUCE_ULIMIT;
+  @Deprecated
+  public static final String MAPRED_REDUCE_TASK_ULIMIT =
+    "mapreduce.reduce.ulimit";
 
 
 
 
   /**
   /**
@@ -1966,6 +1953,15 @@ public class JobConf extends Configuration {
                 + " Instead use " + JobConf.MAPRED_JOB_MAP_MEMORY_MB_PROPERTY
                 + " Instead use " + JobConf.MAPRED_JOB_MAP_MEMORY_MB_PROPERTY
                 + " and " + JobConf.MAPRED_JOB_REDUCE_MEMORY_MB_PROPERTY);
                 + " and " + JobConf.MAPRED_JOB_REDUCE_MEMORY_MB_PROPERTY);
     }
     }
+    if(get(JobConf.MAPRED_TASK_ULIMIT) != null ) {
+      LOG.warn(JobConf.deprecatedString(JobConf.MAPRED_TASK_ULIMIT));
+    }
+    if(get(JobConf.MAPRED_MAP_TASK_ULIMIT) != null ) {
+      LOG.warn(JobConf.deprecatedString(JobConf.MAPRED_MAP_TASK_ULIMIT));
+    }
+    if(get(JobConf.MAPRED_REDUCE_TASK_ULIMIT) != null ) {
+      LOG.warn(JobConf.deprecatedString(JobConf.MAPRED_REDUCE_TASK_ULIMIT));
+    }
   }
   }
   
   
 
 

+ 0 - 4
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapreduce/MRJobConfig.java

@@ -183,8 +183,6 @@ public interface MRJobConfig {
 
 
   public static final String MAP_JAVA_OPTS = "mapreduce.map.java.opts";
   public static final String MAP_JAVA_OPTS = "mapreduce.map.java.opts";
 
 
-  public static final String MAP_ULIMIT = "mapreduce.map.ulimit";
-
   public static final String MAP_MAX_ATTEMPTS = "mapreduce.map.maxattempts";
   public static final String MAP_MAX_ATTEMPTS = "mapreduce.map.maxattempts";
 
 
   public static final String MAP_DEBUG_SCRIPT = "mapreduce.map.debug.script";
   public static final String MAP_DEBUG_SCRIPT = "mapreduce.map.debug.script";
@@ -243,8 +241,6 @@ public interface MRJobConfig {
 
 
   public static final String REDUCE_JAVA_OPTS = "mapreduce.reduce.java.opts";
   public static final String REDUCE_JAVA_OPTS = "mapreduce.reduce.java.opts";
 
 
-  public static final String REDUCE_ULIMIT = "mapreduce.reduce.ulimit";
-  
   public static final String MAPREDUCE_JOB_DIR = "mapreduce.job.dir";
   public static final String MAPREDUCE_JOB_DIR = "mapreduce.job.dir";
 
 
   public static final String REDUCE_MAX_ATTEMPTS = "mapreduce.reduce.maxattempts";
   public static final String REDUCE_MAX_ATTEMPTS = "mapreduce.reduce.maxattempts";

+ 0 - 4
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapreduce/util/ConfigUtil.java

@@ -314,8 +314,6 @@ public class ConfigUtil {
       new String[] {MRJobConfig.MAP_ENV});
       new String[] {MRJobConfig.MAP_ENV});
     Configuration.addDeprecation("mapred.map.child.java.opts", 
     Configuration.addDeprecation("mapred.map.child.java.opts", 
       new String[] {MRJobConfig.MAP_JAVA_OPTS});
       new String[] {MRJobConfig.MAP_JAVA_OPTS});
-    Configuration.addDeprecation("mapred.map.child.ulimit", 
-      new String[] {MRJobConfig.MAP_ULIMIT});
     Configuration.addDeprecation("mapred.map.max.attempts", 
     Configuration.addDeprecation("mapred.map.max.attempts", 
       new String[] {MRJobConfig.MAP_MAX_ATTEMPTS});
       new String[] {MRJobConfig.MAP_MAX_ATTEMPTS});
     Configuration.addDeprecation("mapred.map.task.debug.script", 
     Configuration.addDeprecation("mapred.map.task.debug.script", 
@@ -362,8 +360,6 @@ public class ConfigUtil {
       new String[] {MRJobConfig.REDUCE_ENV});
       new String[] {MRJobConfig.REDUCE_ENV});
     Configuration.addDeprecation("mapred.reduce.child.java.opts", 
     Configuration.addDeprecation("mapred.reduce.child.java.opts", 
       new String[] {MRJobConfig.REDUCE_JAVA_OPTS});
       new String[] {MRJobConfig.REDUCE_JAVA_OPTS});
-    Configuration.addDeprecation("mapred.reduce.child.ulimit", 
-      new String[] {MRJobConfig.REDUCE_ULIMIT});
     Configuration.addDeprecation("mapred.reduce.max.attempts", 
     Configuration.addDeprecation("mapred.reduce.max.attempts", 
       new String[] {MRJobConfig.REDUCE_MAX_ATTEMPTS});
       new String[] {MRJobConfig.REDUCE_MAX_ATTEMPTS});
     Configuration.addDeprecation("mapred.reduce.parallel.copies", 
     Configuration.addDeprecation("mapred.reduce.parallel.copies", 

+ 0 - 17
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/resources/mapred-default.xml

@@ -411,9 +411,6 @@
   /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
   /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
         -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
         -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
   
   
-  The configuration variable mapred.child.ulimit can be used to control the
-  maximum virtual memory of the child processes.
-
   Usage of -Djava.library.path can cause programs to no longer function if
   Usage of -Djava.library.path can cause programs to no longer function if
   hadoop native libraries are used. These values should instead be set as part 
   hadoop native libraries are used. These values should instead be set as part 
   of LD_LIBRARY_PATH in the map / reduce JVM env using the mapreduce.map.env and 
   of LD_LIBRARY_PATH in the map / reduce JVM env using the mapreduce.map.env and 
@@ -431,20 +428,6 @@
   </description>
   </description>
 </property>
 </property>
 
 
-<property>
-  <name>mapred.child.ulimit</name>
-  <value></value>
-  <description>The maximum virtual memory, in KB, of a process launched by the 
-  Map-Reduce framework. This can be used to control both the Mapper/Reducer 
-  tasks and applications using Hadoop Pipes, Hadoop Streaming etc. 
-  By default it is left unspecified to let cluster admins control it via 
-  limits.conf and other such relevant mechanisms.
-  
-  Note: mapred.child.ulimit must be greater than or equal to the -Xmx passed to
-  JavaVM, else the VM might not start. 
-  </description>
-</property>
-
 <property>
 <property>
   <name>mapreduce.admin.user.env</name>
   <name>mapreduce.admin.user.env</name>
   <value>LD_LIBRARY_PATH=$HADOOP_COMMON_HOME/lib/native</value>
   <value>LD_LIBRARY_PATH=$HADOOP_COMMON_HOME/lib/native</value>

+ 0 - 17
hadoop-mapreduce-project/src/java/mapred-default.xml

@@ -403,9 +403,6 @@
   For example, to enable verbose gc logging to a file named for the taskid in
   For example, to enable verbose gc logging to a file named for the taskid in
   /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
   /tmp and to set the heap maximum to be a gigabyte, pass a 'value' of:
         -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
         -Xmx1024m -verbose:gc -Xloggc:/tmp/@taskid@.gc
-  
-  The configuration variable mapred.child.ulimit can be used to control the
-  maximum virtual memory of the child processes. 
   </description>
   </description>
 </property>
 </property>
 
 
@@ -419,20 +416,6 @@
   </description>
   </description>
 </property>
 </property>
 
 
-<property>
-  <name>mapred.child.ulimit</name>
-  <value></value>
-  <description>The maximum virtual memory, in KB, of a process launched by the 
-  Map-Reduce framework. This can be used to control both the Mapper/Reducer 
-  tasks and applications using Hadoop Pipes, Hadoop Streaming etc. 
-  By default it is left unspecified to let cluster admins control it via 
-  limits.conf and other such relevant mechanisms.
-  
-  Note: mapred.child.ulimit must be greater than or equal to the -Xmx passed to
-  JavaVM, else the VM might not start. 
-  </description>
-</property>
-
 <property>
 <property>
   <name>mapreduce.map.log.level</name>
   <name>mapreduce.map.log.level</name>
   <value>INFO</value>
   <value>INFO</value>

+ 0 - 6
hadoop-mapreduce-project/src/java/org/apache/hadoop/mapred/MapTaskRunner.java

@@ -33,12 +33,6 @@ class MapTaskRunner extends TaskRunner {
                        super.getChildJavaOpts(jobConf, 
                        super.getChildJavaOpts(jobConf, 
                            JobConf.DEFAULT_MAPRED_TASK_JAVA_OPTS));
                            JobConf.DEFAULT_MAPRED_TASK_JAVA_OPTS));
   }
   }
-  
-  @Override
-  public int getChildUlimit(JobConf jobConf) {
-    return jobConf.getInt(JobConf.MAPRED_MAP_TASK_ULIMIT, 
-                          super.getChildUlimit(jobConf));
-  }
 
 
   @Override
   @Override
   public String getChildEnv(JobConf jobConf) {
   public String getChildEnv(JobConf jobConf) {

+ 0 - 6
hadoop-mapreduce-project/src/java/org/apache/hadoop/mapred/ReduceTaskRunner.java

@@ -41,12 +41,6 @@ class ReduceTaskRunner extends TaskRunner {
                        super.getChildJavaOpts(jobConf, 
                        super.getChildJavaOpts(jobConf, 
                            JobConf.DEFAULT_MAPRED_TASK_JAVA_OPTS));
                            JobConf.DEFAULT_MAPRED_TASK_JAVA_OPTS));
   }
   }
- 
-  @Override
-  public int getChildUlimit(JobConf jobConf) {
-    return jobConf.getInt(JobConf.MAPRED_REDUCE_TASK_ULIMIT, 
-                          super.getChildUlimit(jobConf));
-  }
 
 
   @Override
   @Override
   public String getChildEnv(JobConf jobConf) {
   public String getChildEnv(JobConf jobConf) {

+ 2 - 36
hadoop-mapreduce-project/src/java/org/apache/hadoop/mapred/TaskRunner.java

@@ -110,20 +110,7 @@ abstract class TaskRunner extends Thread {
   public String getChildJavaOpts(JobConf jobConf, String defaultValue) {
   public String getChildJavaOpts(JobConf jobConf, String defaultValue) {
     return jobConf.get(JobConf.MAPRED_TASK_JAVA_OPTS, defaultValue);
     return jobConf.get(JobConf.MAPRED_TASK_JAVA_OPTS, defaultValue);
   }
   }
-  
-  /**
-   * Get the maximum virtual memory of the child map/reduce tasks.
-   * @param jobConf job configuration
-   * @return the maximum virtual memory of the child task or <code>-1</code> if
-   *         none is specified
-   * @deprecated Use limits specific to the map or reduce tasks set via
-   *             {@link JobConf#MAPRED_MAP_TASK_ULIMIT} or
-   *             {@link JobConf#MAPRED_REDUCE_TASK_ULIMIT} 
-   */
-  @Deprecated
-  public int getChildUlimit(JobConf jobConf) {
-    return jobConf.getInt(JobConf.MAPRED_TASK_ULIMIT, -1);
-  }
+
   
   
   /**
   /**
    * Get the environment variables for the child map/reduce tasks.
    * Get the environment variables for the child map/reduce tasks.
@@ -188,8 +175,7 @@ abstract class TaskRunner extends Thread {
 
 
       tracker.addToMemoryManager(t.getTaskID(), t.isMapTask(), conf);
       tracker.addToMemoryManager(t.getTaskID(), t.isMapTask(), conf);
 
 
-      // set memory limit using ulimit if feasible and necessary ...
-      List<String> setup = getVMSetupCmd();
+      List<String> setup = new ArrayList<String>();
 
 
       // Set up the redirection of the task's stdout and stderr streams
       // Set up the redirection of the task's stdout and stderr streams
       File[] logFiles = prepareLogFiles(taskid, t.isTaskCleanupTask());
       File[] logFiles = prepareLogFiles(taskid, t.isTaskCleanupTask());
@@ -310,26 +296,6 @@ abstract class TaskRunner extends Thread {
     t.setJobFile(localTaskFile.toString());
     t.setJobFile(localTaskFile.toString());
   }
   }
 
 
-  /**
-   * @return
-   */
-  private List<String> getVMSetupCmd() {
-
-    int ulimit = getChildUlimit(conf);
-    if (ulimit <= 0) {
-      return null;
-    }
-    List<String> setup = null;
-    String[] ulimitCmd = Shell.getUlimitMemoryCommand(ulimit);
-    if (ulimitCmd != null) {
-      setup = new ArrayList<String>();
-      for (String arg : ulimitCmd) {
-        setup.add(arg);
-      }
-    }
-    return setup;
-  }
-
   /**
   /**
    * Parse the given string and return an array of individual java opts. Split
    * Parse the given string and return an array of individual java opts. Split
    * on whitespace and replace the special string "@taskid@" with the task ID
    * on whitespace and replace the special string "@taskid@" with the task ID