diff --git a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryInvocationHandler.java b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryInvocationHandler.java index f2b2c9981a..5198c0d27a 100644 --- a/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryInvocationHandler.java +++ b/hadoop-common-project/hadoop-common/src/main/java/org/apache/hadoop/io/retry/RetryInvocationHandler.java @@ -27,6 +27,7 @@ import org.apache.hadoop.ipc.Client.ConnectionId; import java.io.IOException; +import java.io.InterruptedIOException; import java.lang.reflect.InvocationHandler; import java.lang.reflect.InvocationTargetException; import java.lang.reflect.Method; @@ -297,7 +298,16 @@ private void handleException(final Method method, final RetryPolicy policy, log(method, isFailover, counters.failovers, retryInfo.delay, ex); if (retryInfo.delay > 0) { - Thread.sleep(retryInfo.delay); + try { + Thread.sleep(retryInfo.delay); + } catch (InterruptedException e) { + Thread.currentThread().interrupt(); + LOG.warn("Interrupted while waiting to retry", e); + InterruptedIOException intIOE = new InterruptedIOException( + "Retry interrupted"); + intIOE.initCause(e); + throw intIOE; + } } if (isFailover) { diff --git a/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestRetryProxy.java b/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestRetryProxy.java index 41c1be4910..649af89cf1 100644 --- a/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestRetryProxy.java +++ b/hadoop-common-project/hadoop-common/src/test/java/org/apache/hadoop/io/retry/TestRetryProxy.java @@ -31,6 +31,7 @@ import org.mockito.stubbing.Answer; import java.io.IOException; +import java.io.InterruptedIOException; import java.lang.reflect.UndeclaredThrowableException; import java.util.Collections; import java.util.Map; @@ -320,7 +321,9 @@ public Throwable call() throws Exception { futureThread.get().interrupt(); Throwable e = future.get(1, TimeUnit.SECONDS); // should return immediately assertNotNull(e); - assertEquals(InterruptedException.class, e.getClass()); - assertEquals("sleep interrupted", e.getMessage()); + assertEquals(InterruptedIOException.class, e.getClass()); + assertEquals("Retry interrupted", e.getMessage()); + assertEquals(InterruptedException.class, e.getCause().getClass()); + assertEquals("sleep interrupted", e.getCause().getMessage()); } }