浏览代码

MAPREDUCE-3056. svn merge -c r1178740 --ignore-ancestry ../../trunk/

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-0.23@1178741 13f79535-47bb-0310-9956-ffa450edef68
Vinod Kumar Vavilapalli 13 年之前
父节点
当前提交
fd1e8b15d5

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

@@ -1481,6 +1481,9 @@ Release 0.23.0 - Unreleased
     MAPREDUCE-3112. Fixed recursive sourcing of HADOOP_OPTS environment
     variable. (Eric Yang)
 
+    MAPREDUCE-3056. Changed the default staging directory to not include
+    user.name to prevent issues with non-secure mode. (Devaraj K via vinodkv)
+
 Release 0.22.0 - Unreleased
 
   INCOMPATIBLE CHANGES

+ 22 - 6
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-app/src/main/java/org/apache/hadoop/mapreduce/v2/app/MRAppMaster.java

@@ -22,6 +22,7 @@ import java.io.File;
 import java.io.IOException;
 import java.lang.reflect.Constructor;
 import java.lang.reflect.InvocationTargetException;
+import java.security.PrivilegedExceptionAction;
 import java.util.Map;
 import java.util.Set;
 import java.util.concurrent.ConcurrentHashMap;
@@ -656,14 +657,29 @@ public class MRAppMaster extends CompositeService {
           new CompositeServiceShutdownHook(appMaster));
       YarnConfiguration conf = new YarnConfiguration(new JobConf());
       conf.addResource(new Path(MRJobConfig.JOB_CONF_FILE));
-      conf.set(MRJobConfig.USER_NAME, 
-          System.getProperty("user.name")); 
-      UserGroupInformation.setConfiguration(conf);
-      appMaster.init(conf);
-      appMaster.start();
+      String jobUserName = System
+          .getenv(ApplicationConstants.Environment.USER.name());
+      conf.set(MRJobConfig.USER_NAME, jobUserName);
+      initAndStartAppMaster(appMaster, conf, jobUserName);
     } catch (Throwable t) {
       LOG.fatal("Error starting MRAppMaster", t);
       System.exit(1);
     }
-  } 
+  }
+
+  protected static void initAndStartAppMaster(final MRAppMaster appMaster,
+      final YarnConfiguration conf, String jobUserName) throws IOException,
+      InterruptedException {
+    UserGroupInformation.setConfiguration(conf);
+    UserGroupInformation appMasterUgi = UserGroupInformation
+        .createRemoteUser(jobUserName);
+    appMasterUgi.doAs(new PrivilegedExceptionAction<Object>() {
+      @Override
+      public Object run() throws Exception {
+        appMaster.init(conf);
+        appMaster.start();
+        return null;
+      }
+    });
+  }
 }

+ 76 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-app/src/test/java/org/apache/hadoop/mapreduce/v2/app/TestMRAppMaster.java

@@ -0,0 +1,76 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ *
+ *     http://www.apache.org/licenses/LICENSE-2.0
+ *
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+package org.apache.hadoop.mapreduce.v2.app;
+
+import java.io.IOException;
+
+import junit.framework.Assert;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.fs.Path;
+import org.apache.hadoop.mapreduce.MRJobConfig;
+import org.apache.hadoop.mapreduce.v2.util.MRApps;
+import org.apache.hadoop.security.UserGroupInformation;
+import org.apache.hadoop.yarn.api.records.ApplicationAttemptId;
+import org.apache.hadoop.yarn.conf.YarnConfiguration;
+import org.apache.hadoop.yarn.util.ConverterUtils;
+import org.junit.Test;
+
+public class TestMRAppMaster {
+  @Test
+  public void testMRAppMasterForDifferentUser() throws IOException,
+      InterruptedException {
+    String applicationAttemptIdStr = "appattempt_1317529182569_0004_000001";
+    String stagingDir = "/tmp/staging";
+    String userName = "TestAppMasterUser";
+    ApplicationAttemptId applicationAttemptId = ConverterUtils
+        .toApplicationAttemptId(applicationAttemptIdStr);
+    MRAppMasterTest appMaster = new MRAppMasterTest(applicationAttemptId);
+    YarnConfiguration conf = new YarnConfiguration();
+    conf.set(MRJobConfig.MR_AM_STAGING_DIR, stagingDir);
+    MRAppMaster.initAndStartAppMaster(appMaster, conf, userName);
+    Assert.assertEquals(stagingDir + Path.SEPARATOR + userName + Path.SEPARATOR
+        + ".staging", appMaster.stagingDirPath.toString());
+  }
+}
+
+class MRAppMasterTest extends MRAppMaster {
+
+  Path stagingDirPath;
+  private Configuration conf;
+
+  public MRAppMasterTest(ApplicationAttemptId applicationAttemptId) {
+    super(applicationAttemptId);
+  }
+
+  @Override
+  public void init(Configuration conf) {
+    this.conf = conf;
+  }
+
+  @Override
+  public void start() {
+    try {
+      String user = UserGroupInformation.getCurrentUser().getShortUserName();
+      stagingDirPath = MRApps.getStagingAreaDir(conf, user);
+    } catch (Exception e) {
+      Assert.fail(e.getMessage());
+    }
+  }
+
+}

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

@@ -1174,7 +1174,7 @@
 
 <property>
   <name>yarn.app.mapreduce.am.staging-dir</name>
-  <value>/tmp/hadoop-yarn/${user.name}/staging</value>
+  <value>/tmp/hadoop-yarn/staging</value>
   <description>The staging dir used while submitting jobs.
   </description>
 </property>