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 ar...@apache.org on 2015/06/10 05:59:40 UTC

[2/3] hadoop git commit: HADOOP-12078. The default retry policy does not handle RetriableException correctly. (Contributed by Arpit Agarwal)

HADOOP-12078. The default retry policy does not handle RetriableException correctly. (Contributed by Arpit Agarwal)


Project: http://git-wip-us.apache.org/repos/asf/hadoop/repo
Commit: http://git-wip-us.apache.org/repos/asf/hadoop/commit/2d67dff1
Tree: http://git-wip-us.apache.org/repos/asf/hadoop/tree/2d67dff1
Diff: http://git-wip-us.apache.org/repos/asf/hadoop/diff/2d67dff1

Branch: refs/heads/branch-2
Commit: 2d67dff15f548b44af5426c6c11a39ed01fa62d7
Parents: d8d9041
Author: Arpit Agarwal <ar...@apache.org>
Authored: Tue Jun 9 20:58:39 2015 -0700
Committer: Arpit Agarwal <ar...@apache.org>
Committed: Tue Jun 9 20:58:46 2015 -0700

----------------------------------------------------------------------
 hadoop-common-project/hadoop-common/CHANGES.txt |   3 +
 .../apache/hadoop/io/retry/RetryPolicies.java   |   2 +-
 .../org/apache/hadoop/io/retry/RetryUtils.java  |   7 +-
 .../hadoop/io/retry/TestDefaultRetryPolicy.java | 101 +++++++++++++++++++
 .../hdfs/server/namenode/NameNodeRpcServer.java |   3 +-
 5 files changed, 113 insertions(+), 3 deletions(-)
----------------------------------------------------------------------


http://git-wip-us.apache.org/repos/asf/hadoop/blob/2d67dff1/hadoop-common-project/hadoop-common/CHANGES.txt
----------------------------------------------------------------------
diff --git a/hadoop-common-project/hadoop-common/CHANGES.txt b/hadoop-common-project/hadoop-common/CHANGES.txt
index d16262a..06d4873 100644
--- a/hadoop-common-project/hadoop-common/CHANGES.txt
+++ b/hadoop-common-project/hadoop-common/CHANGES.txt
@@ -403,6 +403,9 @@ Release 2.7.1 - UNRELEASED
     HADOOP-12058. Fix dead links to DistCp and Hadoop Archives pages.
     (Kazuho Fujii via aajisaka)
 
+    HADOOP-12078. The default retry policy does not handle RetriableException
+    correctly. (Arpit Agarwal)
+
 Release 2.7.0 - 2015-04-20
 
   INCOMPATIBLE CHANGES

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2d67dff1/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryPolicies.java
----------------------------------------------------------------------
diff --git a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryPolicies.java b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryPolicies.java
index 06dc4cb..d27096f 100644
--- a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryPolicies.java
+++ b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryPolicies.java
@@ -624,7 +624,7 @@ public class RetryPolicies {
     return unwrapped instanceof StandbyException;
   }
   
-  private static RetriableException getWrappedRetriableException(Exception e) {
+  static RetriableException getWrappedRetriableException(Exception e) {
     if (!(e instanceof RemoteException)) {
       return null;
     }

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2d67dff1/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryUtils.java
----------------------------------------------------------------------
diff --git a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryUtils.java b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryUtils.java
index b2e115f..a5a7624 100644
--- a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryUtils.java
+++ b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryUtils.java
@@ -25,6 +25,7 @@ import org.apache.hadoop.conf.Configuration;
 import org.apache.hadoop.ipc.RemoteException;
 
 import com.google.protobuf.ServiceException;
+import org.apache.hadoop.ipc.RetriableException;
 
 public class RetryUtils {
   public static final Log LOG = LogFactory.getLog(RetryUtils.class);
@@ -92,7 +93,11 @@ public class RetryUtils {
 
           //see (1) and (2) in the javadoc of this method.
           final RetryPolicy p;
-          if (e instanceof RemoteException) {
+          if (e instanceof RetriableException
+              || RetryPolicies.getWrappedRetriableException(e) != null) {
+            // RetriableException or RetriableException wrapped
+            p = multipleLinearRandomRetry;
+          } else if (e instanceof RemoteException) {
             final RemoteException re = (RemoteException)e;
             p = remoteExceptionToRetry.equals(re.getClassName())?
                 multipleLinearRandomRetry: RetryPolicies.TRY_ONCE_THEN_FAIL;

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2d67dff1/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestDefaultRetryPolicy.java
----------------------------------------------------------------------
diff --git a/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestDefaultRetryPolicy.java b/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestDefaultRetryPolicy.java
new file mode 100644
index 0000000..8a61c04
--- /dev/null
+++ b/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestDefaultRetryPolicy.java
@@ -0,0 +1,101 @@
+/**
+ * Licensed to the Apache Software Foundation (ASF) under one
+ * or more contributor license agreements.  See the NOTICE file
+ * distributed with this work for additional information
+ * regarding copyright ownership.  The ASF licenses this file
+ * to you under the Apache License, Version 2.0 (the
+ * "License"); you may not use this file except in compliance
+ * with the License.  You may obtain a copy of the License at
+ * <p/>
+ * http://www.apache.org/licenses/LICENSE-2.0
+ * <p/>
+ * Unless required by applicable law or agreed to in writing, software
+ * distributed under the License is distributed on an "AS IS" BASIS,
+ * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ * See the License for the specific language governing permissions and
+ * limitations under the License.
+ */
+
+package org.apache.hadoop.io.retry;
+
+import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.ipc.RemoteException;
+import org.apache.hadoop.ipc.RetriableException;
+import org.junit.Rule;
+import org.junit.Test;
+import org.junit.rules.Timeout;
+
+import java.io.IOException;
+
+import static org.hamcrest.core.Is.is;
+import static org.junit.Assert.assertThat;
+
+/**
+ * Test the behavior of the default retry policy.
+ */
+public class TestDefaultRetryPolicy {
+  @Rule
+  public Timeout timeout = new Timeout(300000);
+
+  /**
+   * Verify that the default retry policy correctly retries
+   * RetriableException when defaultRetryPolicyEnabled is enabled.
+   *
+   * @throws IOException
+   */
+  @Test
+  public void testWithRetriable() throws Exception {
+    Configuration conf = new Configuration();
+    RetryPolicy policy = RetryUtils.getDefaultRetryPolicy(
+        conf, "Test.No.Such.Key",
+        true,                     // defaultRetryPolicyEnabled = true
+        "Test.No.Such.Key", "10000,6",
+        null);
+    RetryPolicy.RetryAction action = policy.shouldRetry(
+        new RetriableException("Dummy exception"), 0, 0, true);
+    assertThat(action.action,
+        is(RetryPolicy.RetryAction.RetryDecision.RETRY));
+  }
+
+  /**
+   * Verify that the default retry policy correctly retries
+   * a RetriableException wrapped in a RemoteException when
+   * defaultRetryPolicyEnabled is enabled.
+   *
+   * @throws IOException
+   */
+  @Test
+  public void testWithWrappedRetriable() throws Exception {
+    Configuration conf = new Configuration();
+    RetryPolicy policy = RetryUtils.getDefaultRetryPolicy(
+        conf, "Test.No.Such.Key",
+        true,                     // defaultRetryPolicyEnabled = true
+        "Test.No.Such.Key", "10000,6",
+        null);
+    RetryPolicy.RetryAction action = policy.shouldRetry(
+        new RemoteException(RetriableException.class.getName(),
+            "Dummy exception"), 0, 0, true);
+    assertThat(action.action,
+        is(RetryPolicy.RetryAction.RetryDecision.RETRY));
+  }
+
+  /**
+   * Verify that the default retry policy does *not* retry
+   * RetriableException when defaultRetryPolicyEnabled is disabled.
+   *
+   * @throws IOException
+   */
+  @Test
+  public void testWithRetriableAndRetryDisabled() throws Exception {
+    Configuration conf = new Configuration();
+    RetryPolicy policy = RetryUtils.getDefaultRetryPolicy(
+        conf, "Test.No.Such.Key",
+        false,                     // defaultRetryPolicyEnabled = false
+        "Test.No.Such.Key", "10000,6",
+        null);
+    RetryPolicy.RetryAction action = policy.shouldRetry(
+        new RetriableException("Dummy exception"), 0, 0, true);
+    assertThat(action.action,
+        is(RetryPolicy.RetryAction.RetryDecision.FAIL));
+  }
+}

http://git-wip-us.apache.org/repos/asf/hadoop/blob/2d67dff1/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/NameNodeRpcServer.java
----------------------------------------------------------------------
diff --git a/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/NameNodeRpcServer.java b/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/NameNodeRpcServer.java
index 5249ffd..cc43d3c 100644
--- a/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/NameNodeRpcServer.java
+++ b/hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/NameNodeRpcServer.java
@@ -144,6 +144,7 @@ import org.apache.hadoop.io.EnumSetWritable;
 import org.apache.hadoop.io.Text;
 import org.apache.hadoop.ipc.ProtobufRpcEngine;
 import org.apache.hadoop.ipc.RPC;
+import org.apache.hadoop.ipc.RetriableException;
 import org.apache.hadoop.ipc.RetryCache;
 import org.apache.hadoop.ipc.RetryCache.CacheEntry;
 import org.apache.hadoop.ipc.RetryCache.CacheEntryWithPayload;
@@ -1880,7 +1881,7 @@ class NameNodeRpcServer implements NamenodeProtocols {
 
   private void checkNNStartup() throws IOException {
     if (!this.nn.isStarted()) {
-      throw new IOException(this.nn.getRole() + " still not started");
+      throw new RetriableException(this.nn.getRole() + " still not started");
     }
   }