浏览代码

HDFS-7494. Checking of closed in DFSInputStream#pread() should be protected by synchronization (Ted Yu via Colin P. McCabe)

Colin Patrick Mccabe 10 年之前
父节点
当前提交
a97a1e7317

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

@@ -598,6 +598,9 @@ Release 2.7.0 - UNRELEASED
     HDFS-6425. Large postponedMisreplicatedBlocks has impact on blockReport
     HDFS-6425. Large postponedMisreplicatedBlocks has impact on blockReport
     latency. (Ming Ma via kihwal)
     latency. (Ming Ma via kihwal)
 
 
+    HDFS-7494. Checking of closed in DFSInputStream#pread() should be protected
+    by synchronization (Ted Yu via Colin P. McCabe)
+
 Release 2.6.1 - UNRELEASED
 Release 2.6.1 - UNRELEASED
 
 
   INCOMPATIBLE CHANGES
   INCOMPATIBLE CHANGES

+ 8 - 7
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/DFSInputStream.java

@@ -41,6 +41,7 @@ import java.util.concurrent.ExecutionException;
 import java.util.concurrent.ExecutorCompletionService;
 import java.util.concurrent.ExecutorCompletionService;
 import java.util.concurrent.Future;
 import java.util.concurrent.Future;
 import java.util.concurrent.TimeUnit;
 import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicBoolean;
 
 
 import org.apache.commons.io.IOUtils;
 import org.apache.commons.io.IOUtils;
 import org.apache.hadoop.classification.InterfaceAudience;
 import org.apache.hadoop.classification.InterfaceAudience;
@@ -90,7 +91,7 @@ implements ByteBufferReadable, CanSetDropBehind, CanSetReadahead,
   public static boolean tcpReadsDisabledForTesting = false;
   public static boolean tcpReadsDisabledForTesting = false;
   private long hedgedReadOpsLoopNumForTesting = 0;
   private long hedgedReadOpsLoopNumForTesting = 0;
   private final DFSClient dfsClient;
   private final DFSClient dfsClient;
-  private boolean closed = false;
+  private AtomicBoolean closed = new AtomicBoolean(false);
   private final String src;
   private final String src;
   private final boolean verifyChecksum;
   private final boolean verifyChecksum;
 
 
@@ -661,7 +662,8 @@ implements ByteBufferReadable, CanSetDropBehind, CanSetReadahead,
    */
    */
   @Override
   @Override
   public synchronized void close() throws IOException {
   public synchronized void close() throws IOException {
-    if (closed) {
+    if (!closed.compareAndSet(false, true)) {
+      DFSClient.LOG.warn("DFSInputStream has been closed already");
       return;
       return;
     }
     }
     dfsClient.checkOpen();
     dfsClient.checkOpen();
@@ -685,7 +687,6 @@ implements ByteBufferReadable, CanSetDropBehind, CanSetReadahead,
       blockReader = null;
       blockReader = null;
     }
     }
     super.close();
     super.close();
-    closed = true;
   }
   }
 
 
   @Override
   @Override
@@ -822,7 +823,7 @@ implements ByteBufferReadable, CanSetDropBehind, CanSetReadahead,
 
 
   private synchronized int readWithStrategy(ReaderStrategy strategy, int off, int len) throws IOException {
   private synchronized int readWithStrategy(ReaderStrategy strategy, int off, int len) throws IOException {
     dfsClient.checkOpen();
     dfsClient.checkOpen();
-    if (closed) {
+    if (closed.get()) {
       throw new IOException("Stream closed");
       throw new IOException("Stream closed");
     }
     }
     Map<ExtendedBlock,Set<DatanodeInfo>> corruptedBlockMap 
     Map<ExtendedBlock,Set<DatanodeInfo>> corruptedBlockMap 
@@ -1375,7 +1376,7 @@ implements ByteBufferReadable, CanSetDropBehind, CanSetReadahead,
       throws IOException {
       throws IOException {
     // sanity checks
     // sanity checks
     dfsClient.checkOpen();
     dfsClient.checkOpen();
-    if (closed) {
+    if (closed.get()) {
       throw new IOException("Stream closed");
       throw new IOException("Stream closed");
     }
     }
     failures = 0;
     failures = 0;
@@ -1484,7 +1485,7 @@ implements ByteBufferReadable, CanSetDropBehind, CanSetReadahead,
     if (targetPos < 0) {
     if (targetPos < 0) {
       throw new EOFException("Cannot seek to negative offset");
       throw new EOFException("Cannot seek to negative offset");
     }
     }
-    if (closed) {
+    if (closed.get()) {
       throw new IOException("Stream is closed!");
       throw new IOException("Stream is closed!");
     }
     }
     boolean done = false;
     boolean done = false;
@@ -1571,7 +1572,7 @@ implements ByteBufferReadable, CanSetDropBehind, CanSetReadahead,
    */
    */
   @Override
   @Override
   public synchronized int available() throws IOException {
   public synchronized int available() throws IOException {
-    if (closed) {
+    if (closed.get()) {
       throw new IOException("Stream closed");
       throw new IOException("Stream closed");
     }
     }