Sfoglia il codice sorgente

HADOOP-3595. Remove deprecated methods for mapred.combine.once
functionality, which was necessary to providing backwards
compatible combiner semantics for 0.18. Contributed by Chris Douglas.


git-svn-id: https://svn.apache.org/repos/asf/hadoop/core/trunk@671219 13f79535-47bb-0310-9956-ffa450edef68

Owen O'Malley 17 anni fa
parent
commit
876616caf6

+ 4 - 0
CHANGES.txt

@@ -4,6 +4,10 @@ Trunk (unreleased changes)
 
   INCOMPATIBLE CHANGES
 
+    HADOOP-3595. Remove deprecated methods for mapred.combine.once 
+    functionality, which was necessary to providing backwards
+    compatible combiner semantics for 0.18. (cdouglas via omalley)
+
   NEW FEATURES
 
   IMPROVEMENTS

+ 0 - 14
src/mapred/org/apache/hadoop/mapred/JobConf.java

@@ -798,20 +798,6 @@ public class JobConf extends Configuration {
     setClass("mapred.combiner.class", theClass, Reducer.class);
   }
   
-  /**
-   * If true, ensures the combiner is run once and only once on output from
-   * the map. Otherwise, combiner may be run zero or more times.
-   */
-  @Deprecated
-  public void setCombineOnceOnly(boolean value) {
-    setBoolean("mapred.combine.once", value);
-  }
-
-  @Deprecated
-  public boolean getCombineOnceOnly() {
-    return getBoolean("mapred.combine.once", false);
-  }
-
   /**
    * Should speculative execution be used for this job? 
    * Defaults to <code>true</code>.

+ 6 - 28
src/mapred/org/apache/hadoop/mapred/MapTask.java

@@ -31,9 +31,7 @@ import java.io.DataOutputStream;
 import java.io.IOException;
 import java.io.OutputStream;
 import java.util.ArrayList;
-import java.util.Iterator;
 import java.util.List;
-import java.util.NoSuchElementException;
 
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
@@ -834,30 +832,11 @@ class MapTask extends Task {
             writer = new IFile.Writer(job, out, keyClass, valClass, codec);
 
             if (i == partition) {
-              if (job.getCombineOnceOnly()) {
-                Reducer combiner =
-                  (Reducer)ReflectionUtils.newInstance(combinerClass, job);
-                combineCollector.setWriter(writer);
-                combiner.reduce(key, new Iterator<V>() {
-                    private boolean done = false;
-                    public boolean hasNext() { return !done; }
-                    public V next() {
-                      if (done)
-                        throw new NoSuchElementException();
-                      done = true;
-                      return value;
-                    }
-                    public void remove() {
-                      throw new UnsupportedOperationException();
-                    }
-                  }, combineCollector, reporter);
-              } else {
-                final long recordStart = out.getPos();
-                writer.append(key, value);
-                // Note that our map byte count will not be accurate with
-                // compression
-                mapOutputByteCounter.increment(out.getPos() - recordStart);
-              }
+              final long recordStart = out.getPos();
+              writer.append(key, value);
+              // Note that our map byte count will not be accurate with
+              // compression
+              mapOutputByteCounter.increment(out.getPos() - recordStart);
             }
             writer.close();
 
@@ -1057,8 +1036,7 @@ class MapTask extends Task {
           segmentStart = finalOut.getPos();
           Writer<K, V> writer = 
               new Writer<K, V>(job, finalOut, keyClass, valClass, codec);
-          if (null == combinerClass || job.getCombineOnceOnly() ||
-              numSpills < minSpillsForCombine) {
+          if (null == combinerClass || numSpills < minSpillsForCombine) {
             Merger.writeFile(kvIter, writer, reporter);
           } else {
             combineCollector.setWriter(writer);

+ 1 - 3
src/mapred/org/apache/hadoop/mapred/ReduceTask.java

@@ -1286,9 +1286,7 @@ class ReduceTask extends Task {
       this.numCopiers = conf.getInt("mapred.reduce.parallel.copies", 5);
       this.maxInFlight = 4 * numCopiers;
       this.maxBackoff = conf.getInt("mapred.reduce.copy.backoff", 300);
-      this.combinerClass = conf.getCombineOnceOnly()
-        ? null
-        : conf.getCombinerClass();
+      this.combinerClass = conf.getCombinerClass();
       combineCollector = (null != combinerClass)
         ? new CombineOutputCollector(reduceCombineOutputCounter)
         : null;