Selaa lähdekoodia

Merging trunk to branch-trunk-win

git-svn-id: https://svn.apache.org/repos/asf/hadoop/common/branches/branch-trunk-win@1422281 13f79535-47bb-0310-9956-ffa450edef68
Suresh Srinivas 12 vuotta sitten
vanhempi
commit
69f75b101d

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

@@ -401,6 +401,9 @@ Release 2.0.3-alpha - Unreleased
     HADOOP-9042. Add a test for umask in FileSystemContractBaseTest.
     (Colin McCabe via eli)
 
+    HADOOP-9127. Update documentation for ZooKeeper Failover Controller.
+    (Daisuke Kobayashi via atm)
+
   OPTIMIZATIONS
 
     HADOOP-8866. SampleQuantiles#query is O(N^2) instead of O(N). (Andrew Wang

+ 66 - 0
hadoop-common-project/hadoop-common/src/main/resources/core-default.xml

@@ -1090,4 +1090,70 @@
   </description>
 </property>
 
+<!-- ha properties -->
+
+<property>
+  <name>ha.health-monitor.connect-retry-interval.ms</name>
+  <value>1000</value>
+  <description>
+    How often to retry connecting to the service.
+  </description>
+</property>
+
+<property>
+  <name>ha.health-monitor.check-interval.ms</name>
+  <value>1000</value>
+  <description>
+    How often to check the service.
+  </description>
+</property>
+
+<property>
+  <name>ha.health-monitor.sleep-after-disconnect.ms</name>
+  <value>1000</value>
+  <description>
+    How long to sleep after an unexpected RPC error.
+  </description>
+</property>
+
+<property>
+  <name>ha.health-monitor.rpc-timeout.ms</name>
+  <value>45000</value>
+  <description>
+    Timeout for the actual monitorHealth() calls.
+  </description>
+</property>
+
+<property>
+  <name>ha.failover-controller.new-active.rpc-timeout.ms</name>
+  <value>60000</value>
+  <description>
+    Timeout that the FC waits for the new active to become active
+  </description>
+</property>
+
+<property>
+  <name>ha.failover-controller.graceful-fence.rpc-timeout.ms</name>
+  <value>5000</value>
+  <description>
+    Timeout that the FC waits for the old active to go to standby
+  </description>
+</property>
+
+<property>
+  <name>ha.failover-controller.graceful-fence.connection.retries</name>
+  <value>1</value>
+  <description>
+    FC connection retries for graceful fencing
+  </description>
+</property>
+
+<property>
+  <name>ha.failover-controller.cli-check.rpc-timeout.ms</name>
+  <value>20000</value>
+  <description>
+    Timeout that the CLI (manual) FC waits for monitorHealth, getServiceState
+  </description>
+</property>
+
 </configuration>

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

@@ -277,6 +277,14 @@ Trunk (Unreleased)
     HADOOP-8957 HDFS tests for AbstractFileSystem#IsValidName should be overridden for
     embedded file systems like ViewFs (Chris Nauroth via Sanjay Radia)
 
+    HDFS-4310. fix test org.apache.hadoop.hdfs.server.datanode.
+    TestStartSecureDataNode (Ivan A. Veselovsky via atm)
+
+    HDFS-4274. BlockPoolSliceScanner does not close verification log during
+    shutdown. (Chris Nauroth via suresh)
+
+    HDFS-4275. MiniDFSCluster-based tests fail on Windows due to failure
+    to delete test namenode directory. (Chris Nauroth via suresh)
 
 Release 2.0.3-alpha - Unreleased 
 
@@ -608,6 +616,9 @@ Release 2.0.3-alpha - Unreleased
     HDFS-2264. NamenodeProtocol has the wrong value for clientPrincipal in
     KerberosInfo annotation. (atm)
 
+    HDFS-4307. SocketCache should use monotonic time. (Colin Patrick McCabe
+    via atm)
+
   BREAKDOWN OF HDFS-3077 SUBTASKS
 
     HDFS-3077. Quorum-based protocol for reading and writing edit logs.

+ 6 - 5
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/SocketCache.java

@@ -37,6 +37,7 @@ import org.apache.hadoop.hdfs.protocol.datatransfer.IOStreamPair;
 import org.apache.hadoop.io.IOUtils;
 import org.apache.hadoop.util.Daemon;
 import org.apache.hadoop.util.StringUtils;
+import org.apache.hadoop.util.Time;
 
 /**
  * A cache of input stream sockets to Data Node.
@@ -53,7 +54,7 @@ class SocketCache {
     public SocketAndStreams(Socket s, IOStreamPair ioStreams) {
       this.sock = s;
       this.ioStreams = ioStreams;
-      this.createTime = System.currentTimeMillis();
+      this.createTime = Time.monotonicNow();
     }
     
     @Override
@@ -205,7 +206,7 @@ class SocketCache {
       Entry<SocketAddress, SocketAndStreams> entry = iter.next();
       // if oldest socket expired, remove it
       if (entry == null || 
-        System.currentTimeMillis() - entry.getValue().getCreateTime() < 
+        Time.monotonicNow() - entry.getValue().getCreateTime() < 
         expiryPeriod) {
         break;
       }
@@ -236,13 +237,13 @@ class SocketCache {
    * older than expiryPeriod minutes
    */
   private void run() throws InterruptedException {
-    for(long lastExpiryTime = System.currentTimeMillis();
+    for(long lastExpiryTime = Time.monotonicNow();
         !Thread.interrupted();
         Thread.sleep(expiryPeriod)) {
-      final long elapsed = System.currentTimeMillis() - lastExpiryTime;
+      final long elapsed = Time.monotonicNow() - lastExpiryTime;
       if (elapsed >= expiryPeriod) {
         evictExpired(expiryPeriod);
-        lastExpiryTime = System.currentTimeMillis();
+        lastExpiryTime = Time.monotonicNow();
       }
     }
     clear();

+ 11 - 1
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/BlockPoolSliceScanner.java

@@ -602,6 +602,15 @@ class BlockPoolSliceScanner {
       lastScanTime.set(Time.now());
     }
   }
+
+  /**
+   * Shuts down this BlockPoolSliceScanner and releases any internal resources.
+   */
+  void shutdown() {
+    if (verificationLog != null) {
+      verificationLog.close();
+    }
+  }
   
   private void scan() {
     if (LOG.isDebugEnabled()) {
@@ -610,7 +619,8 @@ class BlockPoolSliceScanner {
     try {
       adjustThrottler();
         
-      while (datanode.shouldRun && !Thread.interrupted()
+      while (datanode.shouldRun
+          && !datanode.blockScanner.blockScannerThread.isInterrupted()
           && datanode.isBPServiceAlive(blockPoolId)) {
         long now = Time.now();
         synchronized (this) {

+ 19 - 2
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/datanode/DataBlockScanner.java

@@ -100,6 +100,11 @@ public class DataBlockScanner implements Runnable {
       }
       bpScanner.scanBlockPoolSlice();
     }
+
+    // Call shutdown for each allocated BlockPoolSliceScanner.
+    for (BlockPoolSliceScanner bpss: blockPoolScannerMap.values()) {
+      bpss.shutdown();
+    }
   }
 
   // Wait for at least one block pool to be up
@@ -232,9 +237,21 @@ public class DataBlockScanner implements Runnable {
     }
   }
   
-  public synchronized void shutdown() {
+  public void shutdown() {
+    synchronized (this) {
+      if (blockScannerThread != null) {
+        blockScannerThread.interrupt();
+      }
+    }
+
+    // We cannot join within the synchronized block, because it would create a
+    // deadlock situation.  blockScannerThread calls other synchronized methods.
     if (blockScannerThread != null) {
-      blockScannerThread.interrupt();
+      try {
+        blockScannerThread.join();
+      } catch (InterruptedException e) {
+        // shutting down anyway
+      }
     }
   }
 

+ 4 - 1
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/datanode/TestBlockRecovery.java

@@ -97,6 +97,7 @@ public class TestBlockRecovery {
     MiniDFSCluster.getBaseDirectory() + "data";
   private DataNode dn;
   private Configuration conf;
+  private boolean tearDownDone;
   private final static long RECOVERY_ID = 3000L;
   private final static String CLUSTER_ID = "testClusterID";
   private final static String POOL_ID = "BP-TEST";
@@ -121,6 +122,7 @@ public class TestBlockRecovery {
    */
   @Before
   public void startUp() throws IOException {
+    tearDownDone = false;
     conf = new HdfsConfiguration();
     conf.set(DFSConfigKeys.DFS_DATANODE_DATA_DIR_KEY, DATA_DIR);
     conf.set(DFSConfigKeys.DFS_DATANODE_ADDRESS_KEY, "0.0.0.0:0");
@@ -177,7 +179,7 @@ public class TestBlockRecovery {
    */
   @After
   public void tearDown() throws IOException {
-    if (dn != null) {
+    if (!tearDownDone && dn != null) {
       try {
         dn.shutdown();
       } catch(Exception e) {
@@ -188,6 +190,7 @@ public class TestBlockRecovery {
           Assert.assertTrue(
               "Cannot delete data-node dirs", FileUtil.fullyDelete(dir));
       }
+      tearDownDone = true;
     }
   }
 

+ 2 - 12
hadoop-hdfs-project/hadoop-hdfs/src/test/java/org/apache/hadoop/hdfs/server/datanode/TestStartSecureDataNode.java

@@ -17,24 +17,14 @@
 
 package org.apache.hadoop.hdfs.server.datanode;
 
-import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertNotNull;
 import static org.junit.Assert.assertTrue;
-import static org.junit.Assert.fail;
-
-import java.io.IOException;
-import java.security.PrivilegedExceptionAction;
 
 import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.fs.CommonConfigurationKeys;
-import org.apache.hadoop.fs.FileSystem;
-import org.apache.hadoop.fs.Path;
-import org.apache.hadoop.fs.permission.FsPermission;
 import org.apache.hadoop.hdfs.DFSConfigKeys;
 import org.apache.hadoop.hdfs.HdfsConfiguration;
 import org.apache.hadoop.hdfs.MiniDFSCluster;
-import org.apache.hadoop.security.UserGroupInformation;
-import org.apache.hadoop.security.UserGroupInformation.AuthenticationMethod;
 import static org.apache.hadoop.security.SecurityUtilTestHelper.isExternalKdcRunning;
 import org.junit.Assume;
 import org.junit.Before;
@@ -67,7 +57,7 @@ public class TestStartSecureDataNode {
   }
 
   @Test
-  public void testSecureNameNode() throws IOException, InterruptedException {
+  public void testSecureNameNode() throws Exception {
     MiniDFSCluster cluster = null;
     try {
       String nnPrincipal =
@@ -105,9 +95,9 @@ public class TestStartSecureDataNode {
         .build();
       cluster.waitActive();
       assertTrue(cluster.isDataNodeUp());
-
     } catch (Exception ex) {
       ex.printStackTrace();
+      throw ex;
     } finally {
       if (cluster != null) {
         cluster.shutdown();

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

@@ -125,6 +125,9 @@ Release 2.0.3-alpha - Unreleased
 
     YARN-267. Fix fair scheduler web UI. (Sandy Ryza via tomwhite)
 
+    YARN-264. y.s.rm.DelegationTokenRenewer attempts to renew token even 
+    after removing an app. (kkambatl via tucu)
+
 Release 2.0.2-alpha - 2012-09-07 
 
   INCOMPATIBLE CHANGES

+ 12 - 1
hadoop-yarn-project/hadoop-yarn/hadoop-yarn-server/hadoop-yarn-server-resourcemanager/src/main/java/org/apache/hadoop/yarn/server/resourcemanager/security/DelegationTokenRenewer.java

@@ -301,13 +301,18 @@ public class DelegationTokenRenewer extends AbstractService {
    */
   private class RenewalTimerTask extends TimerTask {
     private DelegationTokenToRenew dttr;
+    private boolean cancelled = false;
     
     RenewalTimerTask(DelegationTokenToRenew t) {  
       dttr = t;  
     }
     
     @Override
-    public void run() {
+    public synchronized void run() {
+      if (cancelled) {
+        return;
+      }
+
       Token<?> token = dttr.token;
       try {
         // need to use doAs so that http can find the kerberos tgt
@@ -331,6 +336,12 @@ public class DelegationTokenRenewer extends AbstractService {
         removeFailedDelegationToken(dttr);
       }
     }
+
+    @Override
+    public synchronized boolean cancel() {
+      cancelled = true;
+      return super.cancel();
+    }
   }
   
   /**