Ver Fonte

MAPREDUCE-3527. Fix minor API incompatibilities between 1.0 and 0.23.

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/trunk@1213350 13f79535-47bb-0310-9956-ffa450edef68
Thomas White há 13 anos atrás
pai
commit
6571d39a74

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

@@ -261,6 +261,9 @@ Release 0.23.1 - Unreleased
     MAPREDUCE-3519. Fixed a deadlock in NodeManager LocalDirectories's handling
     service. (Ravi Gummadi via vinodkv)
 
+    MAPREDUCE-3527. Fix minor API incompatibilities between 1.0 and 0.23.
+    (tomwhite)
+
 Release 0.23.0 - 2011-11-01 
 
   INCOMPATIBLE CHANGES

+ 8 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapred/InvalidJobConfException.java

@@ -41,5 +41,13 @@ public class InvalidJobConfException
   public InvalidJobConfException(String msg) {
     super(msg);
   }
+  
+  public InvalidJobConfException(String msg, Throwable t) {
+    super(msg, t);
+  }
+
+  public InvalidJobConfException(Throwable t) {
+    super(t);
+  }
 
 }

+ 10 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapred/JobClient.java

@@ -420,6 +420,16 @@ public class JobClient extends CLI {
     boolean monitorAndPrintJob() throws IOException, InterruptedException {
       return job.monitorAndPrintJob();
     }
+    
+    @Override
+    public String getFailureInfo() throws IOException {
+      try {
+        return job.getStatus().getFailureInfo();
+      } catch (InterruptedException ie) {
+        throw new IOException(ie);
+      }
+    }
+
   }
 
   Cluster cluster;

+ 7 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapred/RunningJob.java

@@ -223,4 +223,11 @@ public interface RunningJob {
    * @throws IOException
    */
   public boolean isRetired() throws IOException;
+  
+  /**
+   * Get failure info for the job.
+   * @return the failure info for the job.
+   * @throws IOException
+   */
+  public String getFailureInfo() throws IOException;
 }

+ 16 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapred/lib/CombineFileInputFormat.java

@@ -23,7 +23,10 @@ import java.util.List;
 
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
+import org.apache.hadoop.fs.FileStatus;
+import org.apache.hadoop.fs.Path;
 import org.apache.hadoop.fs.PathFilter;
+import org.apache.hadoop.mapred.FileInputFormat;
 import org.apache.hadoop.mapred.InputFormat;
 import org.apache.hadoop.mapred.InputSplit;
 import org.apache.hadoop.mapred.JobConf;
@@ -115,5 +118,18 @@ public abstract class CombineFileInputFormat<K, V>
       TaskAttemptContext context) throws IOException {
     return null;
   }
+  
+  /** List input directories.
+   * Subclasses may override to, e.g., select only files matching a regular
+   * expression. 
+   * 
+   * @param job the job to list input paths for
+   * @return array of FileStatus objects
+   * @throws IOException if zero items.
+   */
+  protected FileStatus[] listStatus(JobConf job) throws IOException {
+    List<FileStatus> result = super.listStatus(new Job(job));
+    return result.toArray(new FileStatus[result.size()]);
+  }
 
 }

+ 6 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapreduce/TaskAttemptID.java

@@ -90,6 +90,12 @@ public class TaskAttemptID extends org.apache.hadoop.mapred.ID {
   public TaskID getTaskID() {
     return taskId;
   }
+  
+  /**Returns whether this TaskID is a map ID */
+  @Deprecated
+  public boolean isMap() {
+    return taskId.isMap();
+  }
     
   /**Returns the TaskType of the TaskAttemptID */
   public TaskType getTaskType() {

+ 6 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapreduce/TaskID.java

@@ -100,6 +100,12 @@ public class TaskID extends org.apache.hadoop.mapred.ID {
   public JobID getJobID() {
     return jobId;
   }
+  
+  /**Returns whether this TaskID is a map ID */
+  @Deprecated
+  public boolean isMap() {
+    return type == TaskType.MAP;
+  }
     
   /**
    * Get the type of the task

+ 2 - 0
hadoop-mapreduce-project/hadoop-mapreduce-client/hadoop-mapreduce-client-core/src/main/java/org/apache/hadoop/mapreduce/lib/input/KeyValueLineRecordReader.java

@@ -50,6 +50,8 @@ public class KeyValueLineRecordReader extends RecordReader<Text, Text> {
   
   private Text value;
   
+  public Class getKeyClass() { return Text.class; }
+  
   public KeyValueLineRecordReader(Configuration conf)
     throws IOException {