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 om...@apache.org on 2011/03/04 04:59:55 UTC

svn commit: r1077286 - /hadoop/common/branches/branch-0.20-security-patches/src/hdfs/org/apache/hadoop/hdfs/server/datanode/BlockReceiver.java

Author: omalley
Date: Fri Mar  4 03:59:55 2011
New Revision: 1077286

URL: http://svn.apache.org/viewvc?rev=1077286&view=rev
Log:
commit 02e4668c2f4c571345080c7af0db424db581ee17
Author: Hairong Kuang <ha...@ucdev21.inktomisearch.com>
Date:   Thu Mar 4 19:46:49 2010 +0000

    HDFS:101 from http://issues.apache.org/jira/secure/attachment/12437819/detectDownDN3-0.20-yahoo.patch
    
    +++ b/YAHOO-CHANGES.txt
    +    HDFS-101. DFS write pipeline: DFSClient sometimes does not detect
    +    second datanode failure. (hairong)
    +

Modified:
    hadoop/common/branches/branch-0.20-security-patches/src/hdfs/org/apache/hadoop/hdfs/server/datanode/BlockReceiver.java

Modified: hadoop/common/branches/branch-0.20-security-patches/src/hdfs/org/apache/hadoop/hdfs/server/datanode/BlockReceiver.java
URL: http://svn.apache.org/viewvc/hadoop/common/branches/branch-0.20-security-patches/src/hdfs/org/apache/hadoop/hdfs/server/datanode/BlockReceiver.java?rev=1077286&r1=1077285&r2=1077286&view=diff
==============================================================================
--- hadoop/common/branches/branch-0.20-security-patches/src/hdfs/org/apache/hadoop/hdfs/server/datanode/BlockReceiver.java (original)
+++ hadoop/common/branches/branch-0.20-security-patches/src/hdfs/org/apache/hadoop/hdfs/server/datanode/BlockReceiver.java Fri Mar  4 03:59:55 2011
@@ -75,6 +75,7 @@ class BlockReceiver implements java.io.C
   DatanodeInfo srcDataNode = null;
   private Checksum partialCrc = null;
   private DataNode datanode = null;
+  volatile private boolean mirrorError;
 
   BlockReceiver(Block block, DataInputStream in, String inAddr,
                 String myAddr, boolean isRecovery, String clientName, 
@@ -176,21 +177,19 @@ class BlockReceiver implements java.io.C
 
   /**
    * While writing to mirrorOut, failure to write to mirror should not
-   * affect this datanode unless a client is writing the block.
+   * affect this datanode.
    */
   private void handleMirrorOutError(IOException ioe) throws IOException {
-    LOG.info(datanode.dnRegistration + ":Exception writing block " +
+    LOG.info(datanode.dnRegistration + ": Exception writing block " +
              block + " to mirror " + mirrorAddr + "\n" +
              StringUtils.stringifyException(ioe));
-    mirrorOut = null;
-    //
-    // If stream-copy fails, continue
-    // writing to disk for replication requests. For client
-    // writes, return error so that the client can do error
-    // recovery.
-    //
-    if (clientName.length() > 0) {
+    if (Thread.interrupted()) { // shut down if the thread is interrupted
       throw ioe;
+    } else { // encounter an error while writing to mirror
+      // continue to run even if can not write to mirror
+      // notify client of the error
+      // and wait for the client to shut down the pipeline
+      mirrorError = true;
     }
   }
   
@@ -399,8 +398,8 @@ class BlockReceiver implements java.io.C
     
     setBlockPosition(offsetInBlock);
     
-    //First write the packet to the mirror:
-    if (mirrorOut != null) {
+    // First write the packet to the mirror:
+    if (mirrorOut != null && !mirrorError) {
       try {
         mirrorOut.write(buf.array(), buf.position(), buf.remaining());
         mirrorOut.flush();
@@ -518,7 +517,8 @@ class BlockReceiver implements java.io.C
       if (clientName.length() > 0) {
         responder = new Daemon(datanode.threadGroup, 
                                new PacketResponder(this, block, mirrIn, 
-                                                   replyOut, numTargets));
+                                                   replyOut, numTargets,
+                                                   Thread.currentThread()));
         responder.start(); // start thread to processes reponses
       }
 
@@ -703,18 +703,21 @@ class BlockReceiver implements java.io.C
     DataOutputStream replyOut;  // output to upstream datanode
     private int numTargets;     // number of downstream datanodes including myself
     private BlockReceiver receiver; // The owner of this responder.
+    private Thread receiverThread; // the thread that spawns this responder
 
     public String toString() {
       return "PacketResponder " + numTargets + " for Block " + this.block;
     }
 
     PacketResponder(BlockReceiver receiver, Block b, DataInputStream in, 
-                    DataOutputStream out, int numTargets) {
+                    DataOutputStream out, int numTargets,
+                    Thread receiverThread) {
       this.receiver = receiver;
       this.block = b;
       mirrorIn = in;
       replyOut = out;
       this.numTargets = numTargets;
+      this.receiverThread = receiverThread;
     }
 
     /**
@@ -861,25 +864,27 @@ class BlockReceiver implements java.io.C
       }
 
       boolean lastPacketInBlock = false;
+      boolean isInterrupted = false;
       final long startTime = ClientTraceLog.isInfoEnabled() ? System.nanoTime() : 0;
       while (running && datanode.shouldRun && !lastPacketInBlock) {
 
         try {
-            boolean didRead = false;
             long expected = -2;
             PipelineAck ack = new PipelineAck();
+            long seqno = -2;
             try { 
-              // read an ack from downstream datanode
-              ack.readFields(mirrorIn);
-              if (LOG.isDebugEnabled()) {
-                LOG.debug("PacketResponder " + numTargets + " got " + ack);
-              }
-              long seqno = ack.getSeqno();
-              didRead = true;
+               if (!mirrorError) {
+                // read an ack from downstream datanode
+                 ack.readFields(mirrorIn);
+                 if (LOG.isDebugEnabled()) {
+                   LOG.debug("PacketResponder " + numTargets + " got " + ack);
+                 }
+                 seqno = ack.getSeqno();
+               }
               if (seqno == PipelineAck.HEART_BEAT.getSeqno()) {
                 ack.write(replyOut); // send keepalive
                 replyOut.flush();
-              } else if (seqno >= 0) {
+              } else if (seqno >= 0 || mirrorError) {
                 Packet pkt = null;
                 synchronized (this) {
                   while (running && datanode.shouldRun && ackQueue.size() == 0) {
@@ -891,10 +896,13 @@ class BlockReceiver implements java.io.C
                     }
                     wait();
                   }
+                  if (!running || !datanode.shouldRun) {
+                    break;
+                  }
                   pkt = ackQueue.removeFirst();
                   expected = pkt.seqno;
                   notifyAll();
-                  if (seqno != expected) {
+                  if (seqno != expected && !mirrorError) {
                     throw new IOException("PacketResponder " + numTargets +
                                           " for block " + block +
                                           " expected seqno:" + expected +
@@ -903,27 +911,32 @@ class BlockReceiver implements java.io.C
                   lastPacketInBlock = pkt.lastPacketInBlock;
                 }
               }
-            } catch (Throwable e) {
-              if (running) {
-                LOG.info("PacketResponder " + block + " " + numTargets + 
-                         " Exception " + StringUtils.stringifyException(e));
-                running = false;
+            } catch (InterruptedException ine) {
+              isInterrupted = true;
+            } catch (IOException ioe) {
+              if (Thread.interrupted()) {
+                isInterrupted = true;
+              } else {
+                // continue to run even if can not read from mirror
+                // notify client of the error
+                // and wait for the client to shut down the pipeline
+                mirrorError = true;
+                LOG.info("PacketResponder " + block + " " + numTargets +
+                    " Exception " + StringUtils.stringifyException(ioe));
               }
             }
 
-            if (Thread.interrupted()) {
+            if (Thread.interrupted() || isInterrupted) {
               /* The receiver thread cancelled this thread. 
                * We could also check any other status updates from the 
                * receiver thread (e.g. if it is ok to write to replyOut). 
                * It is prudent to not send any more status back to the client
                * because this datanode has a problem. The upstream datanode
-               * will detect a timout on heartbeats and will declare that
-               * this datanode is bad, and rightly so.
+               * will detect that this datanode is bad, and rightly so.
                */
               LOG.info("PacketResponder " + block +  " " + numTargets +
                        " : Thread is interrupted.");
-              running = false;
-              continue;
+              break;
             }
             
             // If this is the last packet in block, then close block
@@ -952,7 +965,7 @@ class BlockReceiver implements java.io.C
 
             // construct my ack message
             short[] replies = null;
-            if (!didRead) { // no ack is read
+            if (mirrorError) { // no ack is read
               replies = new short[2];
               replies[0] = DataTransferProtocol.OP_STATUS_SUCCESS;
               replies[1] = DataTransferProtocol.OP_STATUS_ERROR;
@@ -973,30 +986,15 @@ class BlockReceiver implements java.io.C
                         " for block " + block +
                         " responded an ack: " + replyAck);
             }
-
-            // If we forwarded an error response from a downstream datanode
-            // and we are acting on behalf of a client, then we quit. The 
-            // client will drive the recovery mechanism.
-            if (!replyAck.isSuccess() && receiver.clientName.length() > 0) {
-              running = false;
-            }
-        } catch (IOException e) {
+        } catch (Throwable e) {
           LOG.warn("IOException in BlockReceiver.run(): ", e);
           if (running) {
-            try {
-              datanode.checkDiskError(e); // may throw an exception here
-            } catch (IOException ioe) {
-              LOG.warn("DataNode.chekDiskError failed in run() with: ", ioe);
-            }
             LOG.info("PacketResponder " + block + " " + numTargets + 
                      " Exception " + StringUtils.stringifyException(e));
             running = false;
           }
-        } catch (RuntimeException e) {
-          if (running) {
-            LOG.info("PacketResponder " + block + " " + numTargets + 
-                     " Exception " + StringUtils.stringifyException(e));
-            running = false;
+          if (!Thread.interrupted()) { // error not caused by interruption
+            receiverThread.interrupt();
           }
         }
       }