瀏覽代碼

HDFS-9797. Log Standby exceptions thrown by RequestHedgingProxyProvider at DEBUG Level (Inigo Goiri via asuresh)

Arun Suresh 9 年之前
父節點
當前提交
9fdfb546fb

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

@@ -1919,6 +1919,9 @@ Release 2.8.0 - UNRELEASED
 
     HDFS-9644. Update encryption documentation to reflect nested EZs. (zhz)
 
+    HDFS-9797. Log Standby exceptions thrown by RequestHedgingProxyProvider
+    at DEBUG Level (Inigo Goiri via asuresh)
+
   OPTIMIZATIONS
 
     HDFS-8026. Trace FSOutputSummer#writeChecksumChunks rather than

+ 39 - 2
hadoop-hdfs-project/hadoop-hdfs/src/main/java/org/apache/hadoop/hdfs/server/namenode/ha/RequestHedgingProxyProvider.java

@@ -17,6 +17,7 @@
  */
 package org.apache.hadoop.hdfs.server.namenode.ha;
 
+import java.io.IOException;
 import java.lang.reflect.InvocationHandler;
 import java.lang.reflect.Method;
 import java.lang.reflect.Proxy;
@@ -33,6 +34,8 @@ import java.util.concurrent.Future;
 import org.apache.commons.logging.Log;
 import org.apache.commons.logging.LogFactory;
 import org.apache.hadoop.conf.Configuration;
+import org.apache.hadoop.ipc.RemoteException;
+import org.apache.hadoop.ipc.StandbyException;
 
 import com.google.common.annotations.VisibleForTesting;
 import org.apache.hadoop.io.retry.MultiException;
@@ -118,8 +121,7 @@ public class RequestHedgingProxyProvider<T> extends
             return retVal;
           } catch (Exception ex) {
             ProxyInfo<T> tProxyInfo = proxyMap.get(callResultFuture);
-            LOG.warn("Invocation returned exception on "
-                    + "[" + tProxyInfo.proxyInfo + "]");
+            logProxyException(ex, tProxyInfo.proxyInfo);
             badResults.put(tProxyInfo.proxyInfo, ex);
             numAttempts--;
           }
@@ -183,4 +185,39 @@ public class RequestHedgingProxyProvider<T> extends
     successfulProxy = null;
   }
 
+  /**
+   * Check the exception returned by the proxy log a warning message if it's
+   * not a StandbyException (expected exception).
+   * @param ex Exception to evaluate.
+   * @param proxyInfo Information of the proxy reporting the exception.
+   */
+  private void logProxyException(Exception ex, String proxyInfo) {
+    if (isStandbyException(ex)) {
+      if (LOG.isDebugEnabled()) {
+        LOG.debug("Invocation returned standby exception on [" +
+            proxyInfo + "]");
+      }
+    } else {
+      LOG.warn("Invocation returned exception on [" + proxyInfo + "]");
+    }
+  }
+
+  /**
+   * Check if the returned exception is caused by an standby namenode.
+   * @param ex Exception to check.
+   * @return If the exception is caused by an standby namenode.
+   */
+  private boolean isStandbyException(Exception ex) {
+    Throwable cause = ex.getCause();
+    if (cause != null) {
+      Throwable cause2 = cause.getCause();
+      if (cause2 instanceof RemoteException) {
+        RemoteException remoteException = (RemoteException)cause2;
+        IOException unwrapRemoteException =
+            remoteException.unwrapRemoteException();
+        return unwrapRemoteException instanceof StandbyException;
+      }
+    }
+    return false;
+  }
 }