|
@@ -17,10 +17,18 @@
|
|
|
*/
|
|
|
package org.apache.hadoop.hdfs;
|
|
|
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY;
|
|
|
import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_PROXY_PROVIDER_KEY_PREFIX;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT;
|
|
|
+import static org.apache.hadoop.hdfs.DFSConfigKeys.DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY;
|
|
|
|
|
|
import java.io.IOException;
|
|
|
import java.lang.reflect.Constructor;
|
|
|
+import java.lang.reflect.InvocationHandler;
|
|
|
+import java.lang.reflect.Proxy;
|
|
|
import java.net.InetSocketAddress;
|
|
|
import java.net.URI;
|
|
|
import java.util.HashMap;
|
|
@@ -48,6 +56,7 @@ import org.apache.hadoop.hdfs.server.protocol.NamenodeProtocols;
|
|
|
import org.apache.hadoop.io.Text;
|
|
|
import org.apache.hadoop.io.retry.DefaultFailoverProxyProvider;
|
|
|
import org.apache.hadoop.io.retry.FailoverProxyProvider;
|
|
|
+import org.apache.hadoop.io.retry.LossyRetryInvocationHandler;
|
|
|
import org.apache.hadoop.io.retry.RetryPolicies;
|
|
|
import org.apache.hadoop.io.retry.RetryPolicy;
|
|
|
import org.apache.hadoop.io.retry.RetryProxy;
|
|
@@ -144,6 +153,61 @@ public class NameNodeProxies {
|
|
|
return new ProxyAndInfo<T>(proxy, dtService);
|
|
|
}
|
|
|
}
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Generate a dummy namenode proxy instance that utilizes our hacked
|
|
|
+ * {@link LossyRetryInvocationHandler}. Proxy instance generated using this
|
|
|
+ * method will proactively drop RPC responses. Currently this method only
|
|
|
+ * support HA setup. IllegalStateException will be thrown if the given
|
|
|
+ * configuration is not for HA.
|
|
|
+ *
|
|
|
+ * @param config the configuration containing the required IPC
|
|
|
+ * properties, client failover configurations, etc.
|
|
|
+ * @param nameNodeUri the URI pointing either to a specific NameNode
|
|
|
+ * or to a logical nameservice.
|
|
|
+ * @param xface the IPC interface which should be created
|
|
|
+ * @param numResponseToDrop The number of responses to drop for each RPC call
|
|
|
+ * @return an object containing both the proxy and the associated
|
|
|
+ * delegation token service it corresponds to
|
|
|
+ * @throws IOException if there is an error creating the proxy
|
|
|
+ */
|
|
|
+ @SuppressWarnings("unchecked")
|
|
|
+ public static <T> ProxyAndInfo<T> createProxyWithLossyRetryHandler(
|
|
|
+ Configuration config, URI nameNodeUri, Class<T> xface,
|
|
|
+ int numResponseToDrop) throws IOException {
|
|
|
+ Preconditions.checkArgument(numResponseToDrop > 0);
|
|
|
+ Class<FailoverProxyProvider<T>> failoverProxyProviderClass =
|
|
|
+ getFailoverProxyProviderClass(config, nameNodeUri, xface);
|
|
|
+ if (failoverProxyProviderClass != null) { // HA case
|
|
|
+ FailoverProxyProvider<T> failoverProxyProvider =
|
|
|
+ createFailoverProxyProvider(config, failoverProxyProviderClass,
|
|
|
+ xface, nameNodeUri);
|
|
|
+ int delay = config.getInt(
|
|
|
+ DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_KEY,
|
|
|
+ DFS_CLIENT_FAILOVER_SLEEPTIME_BASE_DEFAULT);
|
|
|
+ int maxCap = config.getInt(
|
|
|
+ DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_KEY,
|
|
|
+ DFS_CLIENT_FAILOVER_SLEEPTIME_MAX_DEFAULT);
|
|
|
+ int maxFailoverAttempts = config.getInt(
|
|
|
+ DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_KEY,
|
|
|
+ DFS_CLIENT_FAILOVER_MAX_ATTEMPTS_DEFAULT);
|
|
|
+ InvocationHandler dummyHandler = new LossyRetryInvocationHandler<T>(
|
|
|
+ numResponseToDrop, failoverProxyProvider,
|
|
|
+ RetryPolicies.failoverOnNetworkException(
|
|
|
+ RetryPolicies.TRY_ONCE_THEN_FAIL,
|
|
|
+ Math.max(numResponseToDrop + 1, maxFailoverAttempts), delay,
|
|
|
+ maxCap));
|
|
|
+
|
|
|
+ T proxy = (T) Proxy.newProxyInstance(
|
|
|
+ failoverProxyProvider.getInterface().getClassLoader(),
|
|
|
+ new Class[] { xface }, dummyHandler);
|
|
|
+ Text dtService = HAUtil.buildTokenServiceForLogicalUri(nameNodeUri);
|
|
|
+ return new ProxyAndInfo<T>(proxy, dtService);
|
|
|
+ } else {
|
|
|
+ throw new IllegalStateException("Currently creating proxy using " +
|
|
|
+ "LossyRetryInvocationHandler requires NN HA setup");
|
|
|
+ }
|
|
|
+ }
|
|
|
|
|
|
/**
|
|
|
* Creates an explicitly non-HA-enabled proxy object. Most of the time you
|