|
@@ -26,6 +26,8 @@ import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_
|
|
import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_READER_COUNT_KEY;
|
|
import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_READER_COUNT_KEY;
|
|
import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_READER_QUEUE_SIZE_DEFAULT;
|
|
import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_READER_QUEUE_SIZE_DEFAULT;
|
|
import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_READER_QUEUE_SIZE_KEY;
|
|
import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DFS_ROUTER_READER_QUEUE_SIZE_KEY;
|
|
|
|
+import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DN_REPORT_CACHE_EXPIRE;
|
|
|
|
+import static org.apache.hadoop.hdfs.server.federation.router.RBFConfigKeys.DN_REPORT_CACHE_EXPIRE_MS_DEFAULT;
|
|
|
|
|
|
import java.io.FileNotFoundException;
|
|
import java.io.FileNotFoundException;
|
|
import java.io.IOException;
|
|
import java.io.IOException;
|
|
@@ -41,7 +43,19 @@ import java.util.List;
|
|
import java.util.Map;
|
|
import java.util.Map;
|
|
import java.util.Map.Entry;
|
|
import java.util.Map.Entry;
|
|
import java.util.Set;
|
|
import java.util.Set;
|
|
-
|
|
|
|
|
|
+import java.util.concurrent.Callable;
|
|
|
|
+import java.util.concurrent.ExecutionException;
|
|
|
|
+import java.util.concurrent.Executors;
|
|
|
|
+import java.util.concurrent.ThreadFactory;
|
|
|
|
+import java.util.concurrent.TimeUnit;
|
|
|
|
+
|
|
|
|
+import com.google.common.cache.CacheBuilder;
|
|
|
|
+import com.google.common.cache.CacheLoader;
|
|
|
|
+import com.google.common.cache.LoadingCache;
|
|
|
|
+import com.google.common.util.concurrent.ListenableFuture;
|
|
|
|
+import com.google.common.util.concurrent.ListeningExecutorService;
|
|
|
|
+import com.google.common.util.concurrent.MoreExecutors;
|
|
|
|
+import com.google.common.util.concurrent.ThreadFactoryBuilder;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
import org.apache.hadoop.crypto.CryptoProtocolVersion;
|
|
import org.apache.hadoop.crypto.CryptoProtocolVersion;
|
|
import org.apache.hadoop.fs.BatchedRemoteIterator.BatchedEntries;
|
|
import org.apache.hadoop.fs.BatchedRemoteIterator.BatchedEntries;
|
|
@@ -219,6 +233,9 @@ public class RouterRpcServer extends AbstractService implements ClientProtocol,
|
|
private static final ThreadLocal<UserGroupInformation> CUR_USER =
|
|
private static final ThreadLocal<UserGroupInformation> CUR_USER =
|
|
new ThreadLocal<>();
|
|
new ThreadLocal<>();
|
|
|
|
|
|
|
|
+ /** DN type -> full DN report. */
|
|
|
|
+ private final LoadingCache<DatanodeReportType, DatanodeInfo[]> dnCache;
|
|
|
|
+
|
|
/**
|
|
/**
|
|
* Construct a router RPC server.
|
|
* Construct a router RPC server.
|
|
*
|
|
*
|
|
@@ -361,6 +378,23 @@ public class RouterRpcServer extends AbstractService implements ClientProtocol,
|
|
this.nnProto = new RouterNamenodeProtocol(this);
|
|
this.nnProto = new RouterNamenodeProtocol(this);
|
|
this.clientProto = new RouterClientProtocol(conf, this);
|
|
this.clientProto = new RouterClientProtocol(conf, this);
|
|
this.routerProto = new RouterUserProtocol(this);
|
|
this.routerProto = new RouterUserProtocol(this);
|
|
|
|
+
|
|
|
|
+ long dnCacheExpire = conf.getTimeDuration(
|
|
|
|
+ DN_REPORT_CACHE_EXPIRE,
|
|
|
|
+ DN_REPORT_CACHE_EXPIRE_MS_DEFAULT, TimeUnit.MILLISECONDS);
|
|
|
|
+ this.dnCache = CacheBuilder.newBuilder()
|
|
|
|
+ .build(new DatanodeReportCacheLoader());
|
|
|
|
+
|
|
|
|
+ // Actively refresh the dn cache in a configured interval
|
|
|
|
+ Executors
|
|
|
|
+ .newSingleThreadScheduledExecutor()
|
|
|
|
+ .scheduleWithFixedDelay(() -> this.dnCache
|
|
|
|
+ .asMap()
|
|
|
|
+ .keySet()
|
|
|
|
+ .parallelStream()
|
|
|
|
+ .forEach((key) -> this.dnCache.refresh(key)),
|
|
|
|
+ 0,
|
|
|
|
+ dnCacheExpire, TimeUnit.MILLISECONDS);
|
|
}
|
|
}
|
|
|
|
|
|
@Override
|
|
@Override
|
|
@@ -868,6 +902,50 @@ public class RouterRpcServer extends AbstractService implements ClientProtocol,
|
|
return clientProto.getDatanodeReport(type);
|
|
return clientProto.getDatanodeReport(type);
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ /**
|
|
|
|
+ * Get the datanode report from cache.
|
|
|
|
+ *
|
|
|
|
+ * @param type Type of the datanode.
|
|
|
|
+ * @return List of datanodes.
|
|
|
|
+ * @throws IOException If it cannot get the report.
|
|
|
|
+ */
|
|
|
|
+ DatanodeInfo[] getCachedDatanodeReport(DatanodeReportType type)
|
|
|
|
+ throws IOException {
|
|
|
|
+ try {
|
|
|
|
+ DatanodeInfo[] dns = this.dnCache.get(type);
|
|
|
|
+ if (dns == null) {
|
|
|
|
+ LOG.debug("Get null DN report from cache");
|
|
|
|
+ dns = getCachedDatanodeReportImpl(type);
|
|
|
|
+ this.dnCache.put(type, dns);
|
|
|
|
+ }
|
|
|
|
+ return dns;
|
|
|
|
+ } catch (ExecutionException e) {
|
|
|
|
+ LOG.error("Cannot get the DN report for {}", type, e);
|
|
|
|
+ Throwable cause = e.getCause();
|
|
|
|
+ if (cause instanceof IOException) {
|
|
|
|
+ throw (IOException) cause;
|
|
|
|
+ } else {
|
|
|
|
+ throw new IOException(cause);
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ private DatanodeInfo[] getCachedDatanodeReportImpl(
|
|
|
|
+ final DatanodeReportType type) throws IOException {
|
|
|
|
+ // We need to get the DNs as a privileged user
|
|
|
|
+ UserGroupInformation loginUser = UserGroupInformation.getLoginUser();
|
|
|
|
+ RouterRpcServer.setCurrentUser(loginUser);
|
|
|
|
+
|
|
|
|
+ try {
|
|
|
|
+ DatanodeInfo[] dns = clientProto.getDatanodeReport(type);
|
|
|
|
+ LOG.debug("Refresh cached DN report with {} datanodes", dns.length);
|
|
|
|
+ return dns;
|
|
|
|
+ } finally {
|
|
|
|
+ // Reset ugi to remote user for remaining operations.
|
|
|
|
+ RouterRpcServer.resetCurrentUser();
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+
|
|
/**
|
|
/**
|
|
* Get the datanode report with a timeout.
|
|
* Get the datanode report with a timeout.
|
|
* @param type Type of the datanode.
|
|
* @param type Type of the datanode.
|
|
@@ -1748,4 +1826,45 @@ public class RouterRpcServer extends AbstractService implements ClientProtocol,
|
|
public String[] getGroupsForUser(String user) throws IOException {
|
|
public String[] getGroupsForUser(String user) throws IOException {
|
|
return routerProto.getGroupsForUser(user);
|
|
return routerProto.getGroupsForUser(user);
|
|
}
|
|
}
|
|
-}
|
|
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Deals with loading datanode report into the cache and refresh.
|
|
|
|
+ */
|
|
|
|
+ private class DatanodeReportCacheLoader
|
|
|
|
+ extends CacheLoader<DatanodeReportType, DatanodeInfo[]> {
|
|
|
|
+
|
|
|
|
+ private ListeningExecutorService executorService;
|
|
|
|
+
|
|
|
|
+ DatanodeReportCacheLoader() {
|
|
|
|
+ ThreadFactory threadFactory = new ThreadFactoryBuilder()
|
|
|
|
+ .setNameFormat("DatanodeReport-Cache-Reload")
|
|
|
|
+ .setDaemon(true)
|
|
|
|
+ .build();
|
|
|
|
+
|
|
|
|
+ executorService = MoreExecutors.listeningDecorator(
|
|
|
|
+ Executors.newSingleThreadExecutor(threadFactory));
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ @Override
|
|
|
|
+ public DatanodeInfo[] load(DatanodeReportType type) throws Exception {
|
|
|
|
+ return getCachedDatanodeReportImpl(type);
|
|
|
|
+ }
|
|
|
|
+
|
|
|
|
+ /**
|
|
|
|
+ * Override the reload method to provide an asynchronous implementation,
|
|
|
|
+ * so that the query will not be slowed down by the cache refresh. It
|
|
|
|
+ * will return the old cache value and schedule a background refresh.
|
|
|
|
+ */
|
|
|
|
+ @Override
|
|
|
|
+ public ListenableFuture<DatanodeInfo[]> reload(
|
|
|
|
+ final DatanodeReportType type, DatanodeInfo[] oldValue)
|
|
|
|
+ throws Exception {
|
|
|
|
+ return executorService.submit(new Callable<DatanodeInfo[]>() {
|
|
|
|
+ @Override
|
|
|
|
+ public DatanodeInfo[] call() throws Exception {
|
|
|
|
+ return load(type);
|
|
|
|
+ }
|
|
|
|
+ });
|
|
|
|
+ }
|
|
|
|
+ }
|
|
|
|
+}
|