Ver código fonte

HADOOP-8124, HDFS-3034, MAPREDUCE-3956. Remove the deprecated Syncable.sync().

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/trunk@1295999 13f79535-47bb-0310-9956-ffa450edef68
Tsz-wo Sze 13 anos atrás
pai
commit
b2f67b4704

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

@@ -4,6 +4,9 @@ Trunk (unreleased changes)
 
   INCOMPATIBLE CHANGES
 
+    HADOOP-8124. Remove the deprecated FSDataOutputStream constructor,
+    FSDataOutputStream.sync() and Syncable.sync().  (szetszwo)
+
   NEW FEATURES
 
   IMPROVEMENTS

+ 18 - 26
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/FSDataOutputStream.java

@@ -17,7 +17,11 @@
  */
 package org.apache.hadoop.fs;
 
-import java.io.*;
+import java.io.BufferedOutputStream;
+import java.io.DataOutputStream;
+import java.io.FilterOutputStream;
+import java.io.IOException;
+import java.io.OutputStream;
 
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceStability;
@@ -28,20 +32,19 @@ import org.apache.hadoop.classification.InterfaceStability;
 @InterfaceAudience.Public
 @InterfaceStability.Stable
 public class FSDataOutputStream extends DataOutputStream implements Syncable {
-  private OutputStream wrappedStream;
+  private final OutputStream wrappedStream;
 
   private static class PositionCache extends FilterOutputStream {
-    private FileSystem.Statistics statistics;
-    long position;
+    private final FileSystem.Statistics statistics;
+    private long position;
 
-    public PositionCache(OutputStream out, 
-                         FileSystem.Statistics stats,
-                         long pos) throws IOException {
+    PositionCache(OutputStream out, FileSystem.Statistics stats, long pos) {
       super(out);
       statistics = stats;
       position = pos;
     }
 
+    @Override
     public void write(int b) throws IOException {
       out.write(b);
       position++;
@@ -50,6 +53,7 @@ public class FSDataOutputStream extends DataOutputStream implements Syncable {
       }
     }
     
+    @Override
     public void write(byte b[], int off, int len) throws IOException {
       out.write(b, off, len);
       position += len;                            // update position
@@ -58,27 +62,22 @@ public class FSDataOutputStream extends DataOutputStream implements Syncable {
       }
     }
       
-    public long getPos() throws IOException {
+    long getPos() {
       return position;                            // return cached position
     }
-    
+
+    @Override
     public void close() throws IOException {
       out.close();
     }
   }
 
-  @Deprecated
-  public FSDataOutputStream(OutputStream out) throws IOException {
-    this(out, null);
-  }
-
-  public FSDataOutputStream(OutputStream out, FileSystem.Statistics stats)
-    throws IOException {
+  public FSDataOutputStream(OutputStream out, FileSystem.Statistics stats) {
     this(out, stats, 0);
   }
 
   public FSDataOutputStream(OutputStream out, FileSystem.Statistics stats,
-                            long startPosition) throws IOException {
+                            long startPosition) {
     super(new PositionCache(out, stats, startPosition));
     wrappedStream = out;
   }
@@ -88,13 +87,14 @@ public class FSDataOutputStream extends DataOutputStream implements Syncable {
    *
    * @return the current position in the output stream
    */
-  public long getPos() throws IOException {
+  public long getPos() {
     return ((PositionCache)out).getPos();
   }
 
   /**
    * Close the underlying output stream.
    */
+  @Override
   public void close() throws IOException {
     out.close(); // This invokes PositionCache.close()
   }
@@ -109,14 +109,6 @@ public class FSDataOutputStream extends DataOutputStream implements Syncable {
     return wrappedStream;
   }
 
-  @Override  // Syncable
-  @Deprecated
-  public void sync() throws IOException {
-    if (wrappedStream instanceof Syncable) {
-      ((Syncable)wrappedStream).sync();
-    }
-  }
-  
   @Override  // Syncable
   public void hflush() throws IOException {
     if (wrappedStream instanceof Syncable) {

+ 0 - 5
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/fs/Syncable.java

@@ -27,11 +27,6 @@ import org.apache.hadoop.classification.InterfaceStability;
 @InterfaceAudience.Public
 @InterfaceStability.Evolving
 public interface Syncable {
-  /**
-   * @deprecated As of HADOOP 0.21.0, replaced by hflush
-   * @see #hflush()
-   */
-  @Deprecated  public void sync() throws IOException;
   
   /** Flush out the data in client's user buffer. After the return of
    * this call, new readers will see the data.

+ 1 - 1
hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/SequenceFile.java

@@ -1196,7 +1196,7 @@ public class SequenceFile {
     /** flush all currently written data to the file system */
     public void syncFs() throws IOException {
       if (out != null) {
-        out.sync();                               // flush contents to file system
+        out.hflush();  // flush contents to file system
       }
     }
 

+ 2 - 0
hadoop-hdfs-project/hadoop-hdfs/CHANGES.txt

@@ -4,6 +4,8 @@ Trunk (unreleased changes)
 
   INCOMPATIBLE CHANGES
 
+    HDFS-3034. Remove the deprecated DFSOutputStream.sync() method.  (szetszwo)
+
   NEW FEATURES
 
     HDFS-2430. The number of failed or low-resource volumes the NN can tolerate

+ 0 - 6
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSOutputStream.java

@@ -1410,12 +1410,6 @@ class DFSOutputStream extends FSOutputSummer implements Syncable {
       }
     }
   }
-
-  @Override
-  @Deprecated
-  public synchronized void sync() throws IOException {
-    hflush();
-  }
   
   /**
    * Flushes out to all replicas of the block. The data is in the buffers

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

@@ -44,6 +44,9 @@ Trunk (unreleased changes)
 
     MAPREDUCE-2944. Improve checking of input for JobClient.displayTasks() (XieXianshan via harsh)
 
+    MAPREDUCE-3956. Remove the use of the deprecated Syncable.sync() method from
+    TeraOutputFormat in the terasort example.  (szetszwo)
+
   BUG FIXES
 
     MAPREDUCE-3757. [Rumen] Fixed Rumen Folder to adjust shuffleFinished and

+ 1 - 1
hadoop-mapreduce-project/hadoop-mapreduce-examples/src/main/java/org/apache/hadoop/examples/terasort/TeraOutputFormat.java

@@ -71,7 +71,7 @@ public class TeraOutputFormat extends FileOutputFormat<Text,Text> {
     
     public void close(TaskAttemptContext context) throws IOException {
       if (finalSync) {
-        out.sync();
+        out.hsync();
       }
       out.close();
     }