Преглед изворни кода

HADOOP-3986. Remove static Configuration from JobClient. Contributed by Amareshwari Sriramadasu.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/core/trunk@720448 13f79535-47bb-0310-9956-ffa450edef68
Christopher Douglas пре 16 година
родитељ
комит
473fa6526d

+ 6 - 0
CHANGES.txt

@@ -35,6 +35,12 @@ Trunk (unreleased changes)
 
     HADOOP-1650. Upgrade to Jetty 6. (cdouglas)
 
+    HADOOP-3986. Remove static Configuration from JobClient. (Amareshwari
+    Sriramadasu via cdouglas)
+      JobClient::setCommandLineConfig is removed
+      JobClient::getCommandLineConfig is removed
+      JobShell, TestJobShell classes are removed
+
   NEW FEATURES
 
     HADOOP-4575. Add a proxy service for relaying HsftpFileSystem requests.

+ 1 - 1
bin/hadoop

@@ -211,7 +211,7 @@ elif [ "$COMMAND" = "version" ] ; then
   CLASS=org.apache.hadoop.util.VersionInfo
   HADOOP_OPTS="$HADOOP_OPTS $HADOOP_CLIENT_OPTS"
 elif [ "$COMMAND" = "jar" ] ; then
-  CLASS=org.apache.hadoop.mapred.JobShell
+  CLASS=org.apache.hadoop.util.RunJar
 elif [ "$COMMAND" = "distcp" ] ; then
   CLASS=org.apache.hadoop.tools.DistCp
   CLASSPATH=${CLASSPATH}:${TOOL_PATH}

+ 0 - 49
src/mapred/org/apache/hadoop/mapred/JobClient.java

@@ -154,7 +154,6 @@ public class JobClient extends Configured implements MRConstants, Tool  {
   private static final Log LOG = LogFactory.getLog(JobClient.class);
   public static enum TaskStatusFilter { NONE, KILLED, FAILED, SUCCEEDED, ALL }
   private TaskStatusFilter taskOutputFilter = TaskStatusFilter.FAILED; 
-  private static Configuration commandLineConfig;
   static long MAX_JOBPROFILE_AGE = 1000 * 2;
 
   /**
@@ -398,24 +397,6 @@ public class JobClient extends Configured implements MRConstants, Tool  {
     init(conf);
   }
 
-  /**
-   * set the command line config in the jobclient. these are
-   * parameters paassed from the command line and stored in 
-   * conf
-   * @param conf the configuration object to set.
-   */
-  static synchronized void  setCommandLineConfig(Configuration conf) {
-    commandLineConfig = conf;
-  }
-  
-  /**
-   * return the command line configuration
-   */
-  public static synchronized Configuration getCommandLineConfig() {
-    return commandLineConfig;
-  }
-  
- 
   /**
    * Connect to the default {@link JobTracker}.
    * @param conf the job configuration.
@@ -541,8 +522,6 @@ public class JobClient extends Configured implements MRConstants, Tool  {
   private void configureCommandLineOptions(JobConf job, Path submitJobDir, Path submitJarFile) 
     throws IOException {
     
-    final String warning = "Use genericOptions for the option ";
-
     if (!(job.getBoolean("mapred.used.genericoptionsparser", false))) {
       LOG.warn("Use GenericOptionsParser for parsing the arguments. " +
                "Applications should implement Tool for the same.");
@@ -550,41 +529,13 @@ public class JobClient extends Configured implements MRConstants, Tool  {
 
     // get all the command line arguments into the 
     // jobconf passed in by the user conf
-    Configuration commandConf = JobClient.getCommandLineConfig();
     String files = null;
     String libjars = null;
     String archives = null;
 
     files = job.get("tmpfiles");
-    if (files == null) {
-      if (commandConf != null) {
-        files = commandConf.get("tmpfiles");
-        if (files != null) {
-          LOG.warn(warning + "-files");
-        }
-      }
-    }
-
     libjars = job.get("tmpjars");
-    if (libjars == null) {
-      if (commandConf != null) {
-        libjars = commandConf.get("tmpjars");
-        if (libjars != null) {
-          LOG.warn(warning + "-libjars");
-        }
-      }
-    }
-
     archives = job.get("tmparchives");
-    if (archives == null) {
-      if (commandConf != null) {
-        archives = commandConf.get("tmparchives");
-        if (archives != null) {
-          LOG.warn(warning + "-archives");
-        }
-      }
-    }
-    
     /*
      * set this user's id in job configuration, so later job files can be
      * accessed using this user's id

+ 0 - 71
src/mapred/org/apache/hadoop/mapred/JobShell.java

@@ -1,71 +0,0 @@
-/* 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.mapred;
-
-import java.io.IOException;
-
-import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.conf.Configured;
-import org.apache.hadoop.util.RunJar;
-import org.apache.hadoop.util.StringUtils;
-import org.apache.hadoop.util.Tool;
-import org.apache.hadoop.util.ToolRunner;
-
-/** Provide command line parsing for JobSubmission 
- *  job submission looks like 
- *  hadoop jar -libjars <comma seperated jars> -archives <comma seperated archives> 
- *  -files <comma seperated files> inputjar args
- */
-public class JobShell extends Configured implements Tool {
-  public JobShell() {this(null);};
-  
-  public JobShell(Configuration conf) {
-    super(conf);
-  }
-  
-  protected void init() throws IOException {
-    getConf().setQuietMode(false);
-  }
-  
-  /**
-   * run method from Tool
-   */
-  public int run(String argv[]) throws Exception {
-    int exitCode = -1;
-    Configuration conf = getConf();
-    try{
-      JobClient.setCommandLineConfig(conf);
-      try {
-        RunJar.main(argv);
-        exitCode = 0;
-      } catch(Throwable th) {
-        System.err.println(StringUtils.stringifyException(th));
-      }
-    } catch(RuntimeException re) {
-      exitCode = -1;
-      System.err.println(re.getLocalizedMessage());
-    }
-    return exitCode;
-  }
-  
-  public static void main(String[] argv) throws Exception {
-    JobShell jshell = new JobShell();
-    int status = ToolRunner.run(jshell, argv);
-    System.exit(status);
-  }
-}

+ 0 - 79
src/test/org/apache/hadoop/mapred/TestJobShell.java

@@ -1,79 +0,0 @@
-/* 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.mapred;
-
-import java.io.File;
-import java.io.FileOutputStream;
-
-import junit.framework.TestCase;
-
-import org.apache.hadoop.conf.Configuration;
-import org.apache.hadoop.fs.*;
-import org.apache.hadoop.util.ToolRunner;
-import org.apache.hadoop.hdfs.MiniDFSCluster;
-
-public class TestJobShell extends TestCase {
-  // Input output paths for this.. 
-  // these are all dummy and does not test
-  // much in map reduce except for the command line
-  // params 
-  static final Path input = new Path("/test/input/");
-  static final Path output = new Path("/test/output");
-  File buildDir = new File(System.getProperty("test.build.data", "/tmp"));
-  public void testJobShell() throws Exception {
-    MiniDFSCluster dfs = null;
-    MiniMRCluster mr = null;
-    FileSystem fs = null;
-    Path testFile = new Path(input, "testfile");
-    try {
-      Configuration conf = new Configuration();
-      //start the mini mr and dfs cluster.
-      dfs = new MiniDFSCluster(conf, 2 , true, null);
-      fs = dfs.getFileSystem();
-      FSDataOutputStream stream = fs.create(testFile);
-      stream.write("teststring".getBytes());
-      stream.close();
-      mr = new MiniMRCluster(2, fs.getUri().toString(), 1);
-      JobConf jconf = mr.createJobConf();
-      JobShell jshell = new JobShell();
-      File thisbuildDir = new File(buildDir, "jobCommand");
-      assertTrue("create build dir", thisbuildDir.mkdirs()); 
-      File f = new File(thisbuildDir, "files_tmp");
-      FileOutputStream fstream = new FileOutputStream(f);
-      fstream.write("somestrings".getBytes());
-      fstream.close();
-      String[] args = new String[8];
-      args[0] = "-files";
-      args[1] = f.toString();
-      args[2] = "-libjars";
-      /// the testjob.jar as a temporary jar file 
-      // rather than creating its own
-      args[3] = "build/test/testjar/testjob.jar";
-      args[4] = "build/test/testshell/testshell.jar";
-      args[5] = "testshell.ExternalMapReduce";
-      args[6] = input.toString();
-      args[7] = output.toString();
-      int ret = ToolRunner.run(jconf, jshell, args);
-      assertTrue("not failed ", ret != -1);
-      f.delete();
-      thisbuildDir.delete();
-    } finally {
-      if (dfs != null) {dfs.shutdown();};
-      if (mr != null) {mr.shutdown();};
-    }
-  }
-}

+ 2 - 7
src/test/testshell/ExternalMapReduce.java

@@ -125,13 +125,8 @@ public class ExternalMapReduce extends Configured implements Tool {
   }
   
   public static void main(String[] args) throws Exception {
-    Configuration commandConf = JobClient.getCommandLineConfig();
-    if (commandConf != null) {
-      ToolRunner.run(new Configuration(commandConf),
+    int res = ToolRunner.run(new Configuration(),
                      new ExternalMapReduce(), args);
-    } else {
-      ToolRunner.run(new Configuration(),
-                     new ExternalMapReduce(), args);
-    }
+    System.exit(res);
   }
 }