Browse Source

MAPREDUCE-4374. Fix child task environment variable config and add support for Windows. Contributed by Chuan Liu.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/trunk@1502046 13f79535-47bb-0310-9956-ffa450edef68
Chris Nauroth 12 years ago
parent
commit
413bddf596

+ 6 - 0
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/util/Shell.java

@@ -136,6 +136,12 @@ abstract public class Shell {
       new String[] { "kill", "-" + code, isSetsidAvailable ? "-" + pid : pid };
   }
 
+  /** Return a regular expression string that match environment variables */
+  public static String getEnvironmentVariableRegex() {
+    return (WINDOWS) ? "%([A-Za-z_][A-Za-z0-9_]*?)%" :
+      "\\$([A-Za-z_][A-Za-z0-9_]*)";
+  }
+  
   /**
    * Returns a File referencing a script with the given basename, inside the
    * given parent directory.  The file extension is inferred by platform: ".cmd"

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

@@ -624,6 +624,9 @@ Release 2.1.0-beta - 2013-07-02
     MAPREDUCE-5187. Create mapreduce command scripts on Windows. (Chuan Liu via
     cnauroth)
 
+    MAPREDUCE-4374. Fix child task environment variable config and add support
+    for Windows. (Chuan Liu via cnauroth)
+
     MAPREDUCE-5291. Change MR App to use updated property names in
     container-log4j.properties. (Zhijie Shen via sseth)
 

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

@@ -280,12 +280,14 @@ public class JobConf extends Configuration {
    * Configuration key to set the environment of the child map/reduce tasks.
    * 
    * The format of the value is <code>k1=v1,k2=v2</code>. Further it can 
-   * reference existing environment variables via <code>$key</code>.
+   * reference existing environment variables via <code>$key</code> on
+   * Linux or <code>%key%</code> on Windows.
    * 
    * Example:
    * <ul>
    *   <li> A=foo - This will set the env variable A to foo. </li>
-   *   <li> B=$X:c This is inherit tasktracker's X env variable. </li>
+   *   <li> B=$X:c This is inherit tasktracker's X env variable on Linux. </li>
+   *   <li> B=%X%;c This is inherit tasktracker's X env variable on Windows. </li>
    * </ul>
    * 
    * @deprecated Use {@link #MAPRED_MAP_TASK_ENV} or 
@@ -295,31 +297,33 @@ public class JobConf extends Configuration {
   public static final String MAPRED_TASK_ENV = "mapred.child.env";
 
   /**
-   * Configuration key to set the maximum virutal memory available to the
-   * map tasks.
+   * Configuration key to set the environment of the child map tasks.
    * 
-   * The format of the value is <code>k1=v1,k2=v2</code>. Further it can 
-   * reference existing environment variables via <code>$key</code>.
+   * The format of the value is <code>k1=v1,k2=v2</code>. Further it can
+   * reference existing environment variables via <code>$key</code> on
+   * Linux or <code>%key%</code> on Windows.
    * 
    * Example:
    * <ul>
    *   <li> A=foo - This will set the env variable A to foo. </li>
-   *   <li> B=$X:c This is inherit tasktracker's X env variable. </li>
+   *   <li> B=$X:c This is inherit tasktracker's X env variable on Linux. </li>
+   *   <li> B=%X%;c This is inherit tasktracker's X env variable on Windows. </li>
    * </ul>
    */
   public static final String MAPRED_MAP_TASK_ENV = JobContext.MAP_ENV;
   
   /**
-   * Configuration key to set the maximum virutal memory available to the
-   * reduce tasks.
+   * Configuration key to set the environment of the child reduce tasks.
    * 
    * The format of the value is <code>k1=v1,k2=v2</code>. Further it can 
-   * reference existing environment variables via <code>$key</code>.
+   * reference existing environment variables via <code>$key</code> on
+   * Linux or <code>%key%</code> on Windows.
    * 
    * Example:
    * <ul>
    *   <li> A=foo - This will set the env variable A to foo. </li>
-   *   <li> B=$X:c This is inherit tasktracker's X env variable. </li>
+   *   <li> B=$X:c This is inherit tasktracker's X env variable on Linux. </li>
+   *   <li> B=%X%;c This is inherit tasktracker's X env variable on Windows. </li>
    * </ul>
    */
   public static final String MAPRED_REDUCE_TASK_ENV = JobContext.REDUCE_ENV;

+ 2 - 1
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/resources/mapred-default.xml

@@ -176,7 +176,8 @@
   <description>User added environment variables for the task tracker child 
   processes. Example :
   1) A=foo  This will set the env variable A to foo
-  2) B=$B:c This is inherit nodemanager's B env variable.
+  2) B=$B:c This is inherit nodemanager's B env variable on Unix.
+  3) B=%B%;c This is inherit nodemanager's B env variable on Windows.
   </description>
 </property>
 

+ 19 - 12
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-jobclient/src/test/java/org/apache/hadoop/mapred/TestMiniMRChildTask.java

@@ -45,6 +45,7 @@ import org.apache.hadoop.mapreduce.MRJobConfig;
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.hdfs.MiniDFSCluster;
 import org.apache.hadoop.mapreduce.v2.MiniMRYarnCluster;
+import org.apache.hadoop.util.Shell;
 
 /**
  * Class to test mapred task's 
@@ -172,10 +173,10 @@ public class TestMiniMRChildTask {
   private static void checkEnv(String envName, String expValue, String mode) {
     String envValue = System.getenv(envName).trim();
     if ("append".equals(mode)) {
-      if (envValue == null || !envValue.contains(":")) {
+      if (envValue == null || !envValue.contains(File.pathSeparator)) {
         throw new RuntimeException("Missing env variable");
       } else {
-        String parts[] = envValue.split(":");
+        String parts[] = envValue.split(File.pathSeparator);
         // check if the value is appended
         if (!parts[parts.length - 1].equals(expValue)) {
           throw new RuntimeException("Wrong env variable in append mode");
@@ -225,10 +226,10 @@ public class TestMiniMRChildTask {
       // check if X=/tmp for a new env variable
       checkEnv("MY_PATH", "/tmp", "noappend");
       // check if X=$X:/tmp works for a new env var and results into :/tmp
-      checkEnv("NEW_PATH", ":/tmp", "noappend");
+      checkEnv("NEW_PATH", File.pathSeparator + "/tmp", "noappend");
       // check if X=$(tt's X var):/tmp for an old env variable inherited from 
       // the tt
-      checkEnv("PATH",  path + ":/tmp", "noappend");
+      checkEnv("PATH",  path + File.pathSeparator + "/tmp", "noappend");
 
       String jobLocalDir = job.get(MRJobConfig.JOB_LOCAL_DIR);
       assertNotNull(MRJobConfig.JOB_LOCAL_DIR + " is null",
@@ -279,10 +280,10 @@ public class TestMiniMRChildTask {
       // check if X=/tmp for a new env variable
       checkEnv("MY_PATH", "/tmp", "noappend");
       // check if X=$X:/tmp works for a new env var and results into :/tmp
-      checkEnv("NEW_PATH", ":/tmp", "noappend");
+      checkEnv("NEW_PATH", File.pathSeparator + "/tmp", "noappend");
       // check if X=$(tt's X var):/tmp for an old env variable inherited from 
       // the tt
-      checkEnv("PATH",  path + ":/tmp", "noappend");
+      checkEnv("PATH",  path + File.pathSeparator + "/tmp", "noappend");
 
     }
 
@@ -437,12 +438,18 @@ public class TestMiniMRChildTask {
       mapTaskJavaOptsKey = reduceTaskJavaOptsKey = JobConf.MAPRED_TASK_JAVA_OPTS;
       mapTaskJavaOpts = reduceTaskJavaOpts = TASK_OPTS_VAL;
     }
-    conf.set(mapTaskEnvKey, 
-             "MY_PATH=/tmp,LANG=en_us_8859_1,LD_LIBRARY_PATH=$LD_LIBRARY_PATH:/tmp," +
-             "PATH=$PATH:/tmp,NEW_PATH=$NEW_PATH:/tmp");
-    conf.set(reduceTaskEnvKey, 
-             "MY_PATH=/tmp,LANG=en_us_8859_1,LD_LIBRARY_PATH=$LD_LIBRARY_PATH:/tmp," +
-             "PATH=$PATH:/tmp,NEW_PATH=$NEW_PATH:/tmp");
+    conf.set(
+        mapTaskEnvKey,
+        Shell.WINDOWS ? "MY_PATH=/tmp,LANG=en_us_8859_1,LD_LIBRARY_PATH=%LD_LIBRARY_PATH%;/tmp,"
+            + "PATH=%PATH%;/tmp,NEW_PATH=%NEW_PATH%;/tmp"
+            : "MY_PATH=/tmp,LANG=en_us_8859_1,LD_LIBRARY_PATH=$LD_LIBRARY_PATH:/tmp,"
+                + "PATH=$PATH:/tmp,NEW_PATH=$NEW_PATH:/tmp");
+    conf.set(
+        reduceTaskEnvKey,
+        Shell.WINDOWS ? "MY_PATH=/tmp,LANG=en_us_8859_1,LD_LIBRARY_PATH=%LD_LIBRARY_PATH%;/tmp,"
+            + "PATH=%PATH%;/tmp,NEW_PATH=%NEW_PATH%;/tmp"
+            : "MY_PATH=/tmp,LANG=en_us_8859_1,LD_LIBRARY_PATH=$LD_LIBRARY_PATH:/tmp,"
+                + "PATH=$PATH:/tmp,NEW_PATH=$NEW_PATH:/tmp");
     conf.set("path", System.getenv("PATH"));
     conf.set(mapTaskJavaOptsKey, mapTaskJavaOpts);
     conf.set(reduceTaskJavaOptsKey, reduceTaskJavaOpts);

+ 22 - 29
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-common/src/main/java/org/apache/hadoop/yarn/util/Apps.java

@@ -22,12 +22,16 @@ import static org.apache.hadoop.yarn.util.StringHelper._split;
 import static org.apache.hadoop.yarn.util.StringHelper.join;
 import static org.apache.hadoop.yarn.util.StringHelper.sjoin;
 
+import java.io.File;
 import java.util.Iterator;
 import java.util.Map;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
 
 import org.apache.hadoop.classification.InterfaceAudience.Private;
 import org.apache.hadoop.classification.InterfaceAudience.Public;
 import org.apache.hadoop.classification.InterfaceStability.Unstable;
+import org.apache.hadoop.util.Shell;
 import org.apache.hadoop.util.StringInterner;
 import org.apache.hadoop.yarn.api.records.ApplicationId;
 import org.apache.hadoop.yarn.exceptions.YarnRuntimeException;
@@ -69,41 +73,30 @@ public class Apps {
       String envString) {
     if (envString != null && envString.length() > 0) {
       String childEnvs[] = envString.split(",");
+      Pattern p = Pattern.compile(Shell.getEnvironmentVariableRegex());
       for (String cEnv : childEnvs) {
         String[] parts = cEnv.split("="); // split on '='
-        String value = env.get(parts[0]);
-
-        if (value != null) {
-          // Replace $env with the child's env constructed by NM's
-          // For example: LD_LIBRARY_PATH=$LD_LIBRARY_PATH:/tmp
-          value = parts[1].replace("$" + parts[0], value);
-        } else {
-          // example PATH=$PATH:/tmp
-          value = System.getenv(parts[0]);
-          if (value != null) {
-            // the env key is present in the tt's env
-            value = parts[1].replace("$" + parts[0], value);
-          } else {
-            // check for simple variable substitution
-            // for e.g. ROOT=$HOME
-            String envValue = System.getenv(parts[1].substring(1));
-            if (envValue != null) {
-              value = envValue;
-            } else {
-              // the env key is note present anywhere .. simply set it
-              // example X=$X:/tmp or X=/tmp
-              value = parts[1].replace("$" + parts[0], "");
-            }
-          }
+        Matcher m = p.matcher(parts[1]);
+        StringBuffer sb = new StringBuffer();
+        while (m.find()) {
+          String var = m.group(1);
+          // replace $env with the child's env constructed by tt's
+          String replace = env.get(var);
+          // if this key is not configured by the tt for the child .. get it
+          // from the tt's env
+          if (replace == null)
+            replace = System.getenv(var);
+          // the env key is note present anywhere .. simply set it
+          if (replace == null)
+            replace = "";
+          m.appendReplacement(sb, Matcher.quoteReplacement(replace));
         }
-        addToEnvironment(env, parts[0], value);
+        m.appendTail(sb);
+        addToEnvironment(env, parts[0], sb.toString());
       }
     }
   }
 
-  private static final String SYSTEM_PATH_SEPARATOR =
-      System.getProperty("path.separator");
-
   @Public
   @Unstable
   public static void addToEnvironment(
@@ -113,7 +106,7 @@ public class Apps {
     if (val == null) {
       val = value;
     } else {
-      val = val + SYSTEM_PATH_SEPARATOR + value;
+      val = val + File.pathSeparator + value;
     }
     environment.put(StringInterner.weakIntern(variable), 
         StringInterner.weakIntern(val));