You are viewing a plain text version of this content. The canonical link for it is here.
Posted to common-commits@hadoop.apache.org by ac...@apache.org on 2012/03/29 09:51:00 UTC

svn commit: r1306741 - in /hadoop/common/branches/branch-1: CHANGES.txt src/mapred/org/apache/hadoop/mapred/JobClient.java src/test/org/apache/hadoop/mapreduce/TestMROutputFormat.java

Author: acmurthy
Date: Thu Mar 29 07:51:00 2012
New Revision: 1306741

URL: http://svn.apache.org/viewvc?rev=1306741&view=rev
Log:
Ensure OutputCommitter.checkOutputSpecs is called prior to copying job.xml. Contributed by Jane Chen.

Added:
    hadoop/common/branches/branch-1/src/test/org/apache/hadoop/mapreduce/TestMROutputFormat.java
Modified:
    hadoop/common/branches/branch-1/CHANGES.txt
    hadoop/common/branches/branch-1/src/mapred/org/apache/hadoop/mapred/JobClient.java

Modified: hadoop/common/branches/branch-1/CHANGES.txt
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-1/CHANGES.txt?rev=1306741&r1=1306740&r2=1306741&view=diff
==============================================================================
--- hadoop/common/branches/branch-1/CHANGES.txt (original)
+++ hadoop/common/branches/branch-1/CHANGES.txt Thu Mar 29 07:51:00 2012
@@ -202,6 +202,9 @@ Release 1.0.3 - unreleased
     HDFS-3127. Do not throw exceptions when FSImage.restoreStorageDirs() failes.
     (Brandon Li via szetszwo)
 
+    MAPREDUCE-3377. Ensure OutputCommitter.checkOutputSpecs is called prior to
+    copying job.xml. (Jane Chen via acmurthy)
+
 Release 1.0.2 - 2012.03.24
 
   NEW FEATURES

Modified: hadoop/common/branches/branch-1/src/mapred/org/apache/hadoop/mapred/JobClient.java
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-1/src/mapred/org/apache/hadoop/mapred/JobClient.java?rev=1306741&r1=1306740&r2=1306741&view=diff
==============================================================================
--- hadoop/common/branches/branch-1/src/mapred/org/apache/hadoop/mapred/JobClient.java (original)
+++ hadoop/common/branches/branch-1/src/mapred/org/apache/hadoop/mapred/JobClient.java Thu Mar 29 07:51:00 2012
@@ -878,8 +878,6 @@ public class JobClient extends Configure
           }
           JobContext context = new JobContext(jobCopy, jobId);
 
-          jobCopy = (JobConf)context.getConfiguration();
-
           // Check the output specification
           if (reduces == 0 ? jobCopy.getUseNewMapper() : 
             jobCopy.getUseNewReducer()) {
@@ -890,6 +888,8 @@ public class JobClient extends Configure
           } else {
             jobCopy.getOutputFormat().checkOutputSpecs(fs, jobCopy);
           }
+          
+          jobCopy = (JobConf)context.getConfiguration();
 
           // Create the splits for the job
           FileSystem fs = submitJobDir.getFileSystem(jobCopy);

Added: hadoop/common/branches/branch-1/src/test/org/apache/hadoop/mapreduce/TestMROutputFormat.java
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-1/src/test/org/apache/hadoop/mapreduce/TestMROutputFormat.java?rev=1306741&view=auto
==============================================================================
--- hadoop/common/branches/branch-1/src/test/org/apache/hadoop/mapreduce/TestMROutputFormat.java (added)
+++ hadoop/common/branches/branch-1/src/test/org/apache/hadoop/mapreduce/TestMROutputFormat.java Thu Mar 29 07:51:00 2012
@@ -0,0 +1,188 @@
+package org.apache.hadoop.mapreduce;
+
+import java.io.DataInput;
+import java.io.DataOutput;
+import java.io.IOException;
+import java.util.ArrayList;
+import java.util.List;
+
+import org.apache.hadoop.conf.Configurable;
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.io.IntWritable;
+import org.apache.hadoop.io.Writable;
+import org.apache.hadoop.mapred.JobConf;
+import org.junit.Test;
+
+import static org.junit.Assert.assertTrue;
+
+public class TestMROutputFormat {
+
+  @Test
+  public void testJobSubmission() throws Exception {
+    JobConf conf = new JobConf();
+    Job job = new Job(conf);
+    job.setInputFormatClass(TestInputFormat.class);
+    job.setMapperClass(TestMapper.class);
+    job.setOutputFormatClass(TestOutputFormat.class);
+    job.setOutputKeyClass(IntWritable.class);
+    job.setOutputValueClass(IntWritable.class);
+    job.waitForCompletion(true);
+    assertTrue(job.isSuccessful());
+  }
+  
+  public static class TestMapper
+  extends Mapper<IntWritable, IntWritable, IntWritable, IntWritable> {
+    public void map(IntWritable key, IntWritable value, Context context) 
+    throws IOException, InterruptedException {
+      context.write(key, value);
+    }
+  }
+}
+
+class TestInputFormat extends InputFormat<IntWritable, IntWritable> {
+
+  @Override
+  public RecordReader<IntWritable, IntWritable> createRecordReader(
+      InputSplit split, TaskAttemptContext context) throws IOException,
+      InterruptedException {
+    return new RecordReader<IntWritable, IntWritable>() {
+
+      private boolean done = false;
+      
+      @Override
+      public void close() throws IOException {
+      }
+
+      @Override
+      public IntWritable getCurrentKey() throws IOException,
+          InterruptedException {
+	return new IntWritable(0);
+      }
+
+      @Override
+      public IntWritable getCurrentValue() throws IOException,
+          InterruptedException {
+	return new IntWritable(0);
+      }
+
+      @Override
+      public float getProgress() throws IOException, InterruptedException {
+	return done ? 0 : 1;
+      }
+
+      @Override
+      public void initialize(InputSplit split, TaskAttemptContext context)
+          throws IOException, InterruptedException {
+      }
+
+      @Override
+      public boolean nextKeyValue() throws IOException, InterruptedException {
+	if (!done) {
+	  done = true;
+	  return true;
+	}
+	return false;
+      }
+    };
+  }
+
+  @Override
+  public List<InputSplit> getSplits(JobContext context) throws IOException,
+      InterruptedException {    
+    List<InputSplit> list = new ArrayList<InputSplit>();
+    list.add(new TestInputSplit());
+    return list;
+  }
+}
+
+class TestInputSplit extends InputSplit implements Writable {
+  
+  @Override
+  public long getLength() throws IOException, InterruptedException {
+	return 1;
+  }
+
+  @Override
+  public String[] getLocations() throws IOException, InterruptedException {
+	String[] hosts = {"localhost"};
+	return hosts;
+  }
+
+  @Override
+  public void readFields(DataInput in) throws IOException {
+  }
+
+  @Override
+  public void write(DataOutput out) throws IOException {
+  }	
+}
+
+class TestOutputFormat extends OutputFormat<IntWritable, IntWritable> 
+implements Configurable {
+
+  public static final String TEST_CONFIG_NAME = "mapred.test.jobsubmission";
+  private Configuration conf;
+  
+  @Override
+  public void checkOutputSpecs(JobContext context) throws IOException,
+      InterruptedException {
+    conf.setBoolean(TEST_CONFIG_NAME, true);
+  }
+
+  @Override
+  public OutputCommitter getOutputCommitter(TaskAttemptContext context)
+      throws IOException, InterruptedException {
+    return new OutputCommitter() {
+
+      @Override
+      public void abortTask(TaskAttemptContext taskContext) throws IOException {
+      }
+
+      @Override
+      public void commitTask(TaskAttemptContext taskContext) throws IOException {
+      }
+
+      @Override
+      public boolean needsTaskCommit(TaskAttemptContext taskContext)
+          throws IOException {
+	return false;
+      }
+
+      @Override
+      public void setupJob(JobContext jobContext) throws IOException {
+      }
+
+      @Override
+      public void setupTask(TaskAttemptContext taskContext) throws IOException {
+      }
+    };
+  }
+
+  @Override
+  public RecordWriter<IntWritable, IntWritable> getRecordWriter(
+      TaskAttemptContext context) throws IOException, InterruptedException {
+    assertTrue(context.getConfiguration().getBoolean(TEST_CONFIG_NAME, false));
+    return new RecordWriter<IntWritable, IntWritable>() {
+
+      @Override
+      public void close(TaskAttemptContext context) throws IOException,
+          InterruptedException {	
+      }
+
+      @Override
+      public void write(IntWritable key, IntWritable value) throws IOException,
+          InterruptedException {	
+      }
+    }; 
+  }
+  
+  @Override
+  public Configuration getConf() {
+      return conf;
+  }
+
+  @Override
+  public void setConf(Configuration conf) {
+      this.conf = conf;        
+  }
+}
\ No newline at end of file