|
@@ -20,8 +20,9 @@ package org.apache.hadoop.scm;
|
|
|
|
|
|
import java.io.IOException;
|
|
import java.io.IOException;
|
|
import java.util.concurrent.TimeUnit;
|
|
import java.util.concurrent.TimeUnit;
|
|
-import java.util.concurrent.atomic.AtomicInteger;
|
|
|
|
|
|
+import java.util.concurrent.Callable;
|
|
|
|
|
|
|
|
+import com.google.common.annotations.VisibleForTesting;
|
|
import com.google.common.base.Preconditions;
|
|
import com.google.common.base.Preconditions;
|
|
|
|
|
|
import com.google.common.cache.Cache;
|
|
import com.google.common.cache.Cache;
|
|
@@ -31,8 +32,14 @@ import com.google.common.cache.RemovalNotification;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
import org.apache.hadoop.conf.Configuration;
|
|
import org.apache.hadoop.scm.container.common.helpers.Pipeline;
|
|
import org.apache.hadoop.scm.container.common.helpers.Pipeline;
|
|
|
|
|
|
-import static org.apache.hadoop.scm.ScmConfigKeys.SCM_CONTAINER_CLIENT_STALE_THRESHOLD_DEFAULT;
|
|
|
|
-import static org.apache.hadoop.scm.ScmConfigKeys.SCM_CONTAINER_CLIENT_STALE_THRESHOLD_KEY;
|
|
|
|
|
|
+import static org.apache.hadoop.scm.ScmConfigKeys
|
|
|
|
+ .SCM_CONTAINER_CLIENT_STALE_THRESHOLD_DEFAULT;
|
|
|
|
+import static org.apache.hadoop.scm.ScmConfigKeys
|
|
|
|
+ .SCM_CONTAINER_CLIENT_STALE_THRESHOLD_KEY;
|
|
|
|
+import static org.apache.hadoop.scm.ScmConfigKeys
|
|
|
|
+ .SCM_CONTAINER_CLIENT_MAX_SIZE_KEY;
|
|
|
|
+import static org.apache.hadoop.scm.ScmConfigKeys
|
|
|
|
+ .SCM_CONTAINER_CLIENT_MAX_SIZE_DEFAULT;
|
|
|
|
|
|
/**
|
|
/**
|
|
* XceiverClientManager is responsible for the lifecycle of XceiverClient
|
|
* XceiverClientManager is responsible for the lifecycle of XceiverClient
|
|
@@ -50,8 +57,7 @@ public class XceiverClientManager {
|
|
|
|
|
|
//TODO : change this to SCM configuration class
|
|
//TODO : change this to SCM configuration class
|
|
private final Configuration conf;
|
|
private final Configuration conf;
|
|
- private Cache<String, XceiverClientWithAccessInfo> openClient;
|
|
|
|
- private final long staleThresholdMs;
|
|
|
|
|
|
+ private final Cache<String, XceiverClientSpi> clientCache;
|
|
private final boolean useRatis;
|
|
private final boolean useRatis;
|
|
|
|
|
|
/**
|
|
/**
|
|
@@ -61,121 +67,91 @@ public class XceiverClientManager {
|
|
*/
|
|
*/
|
|
public XceiverClientManager(Configuration conf) {
|
|
public XceiverClientManager(Configuration conf) {
|
|
Preconditions.checkNotNull(conf);
|
|
Preconditions.checkNotNull(conf);
|
|
- this.staleThresholdMs = conf.getTimeDuration(
|
|
|
|
|
|
+ int maxSize = conf.getInt(SCM_CONTAINER_CLIENT_MAX_SIZE_KEY,
|
|
|
|
+ SCM_CONTAINER_CLIENT_MAX_SIZE_DEFAULT);
|
|
|
|
+ long staleThresholdMs = conf.getTimeDuration(
|
|
SCM_CONTAINER_CLIENT_STALE_THRESHOLD_KEY,
|
|
SCM_CONTAINER_CLIENT_STALE_THRESHOLD_KEY,
|
|
SCM_CONTAINER_CLIENT_STALE_THRESHOLD_DEFAULT, TimeUnit.MILLISECONDS);
|
|
SCM_CONTAINER_CLIENT_STALE_THRESHOLD_DEFAULT, TimeUnit.MILLISECONDS);
|
|
this.useRatis = conf.getBoolean(
|
|
this.useRatis = conf.getBoolean(
|
|
ScmConfigKeys.DFS_CONTAINER_RATIS_ENABLED_KEY,
|
|
ScmConfigKeys.DFS_CONTAINER_RATIS_ENABLED_KEY,
|
|
ScmConfigKeys.DFS_CONTAINER_RATIS_ENABLED_DEFAULT);
|
|
ScmConfigKeys.DFS_CONTAINER_RATIS_ENABLED_DEFAULT);
|
|
this.conf = conf;
|
|
this.conf = conf;
|
|
- this.openClient = CacheBuilder.newBuilder()
|
|
|
|
- .expireAfterAccess(this.staleThresholdMs, TimeUnit.MILLISECONDS)
|
|
|
|
|
|
+ this.clientCache = CacheBuilder.newBuilder()
|
|
|
|
+ .expireAfterAccess(staleThresholdMs, TimeUnit.MILLISECONDS)
|
|
|
|
+ .maximumSize(maxSize)
|
|
.removalListener(
|
|
.removalListener(
|
|
- new RemovalListener<String, XceiverClientWithAccessInfo>() {
|
|
|
|
|
|
+ new RemovalListener<String, XceiverClientSpi>() {
|
|
@Override
|
|
@Override
|
|
public void onRemoval(
|
|
public void onRemoval(
|
|
- RemovalNotification<String, XceiverClientWithAccessInfo>
|
|
|
|
|
|
+ RemovalNotification<String, XceiverClientSpi>
|
|
removalNotification) {
|
|
removalNotification) {
|
|
- // If the reference count is not 0, this xceiver client should not
|
|
|
|
- // be evicted, add it back to the cache.
|
|
|
|
- XceiverClientWithAccessInfo info = removalNotification.getValue();
|
|
|
|
- if (info.hasRefence()) {
|
|
|
|
- synchronized (XceiverClientManager.this.openClient) {
|
|
|
|
- XceiverClientManager.this
|
|
|
|
- .openClient.put(removalNotification.getKey(), info);
|
|
|
|
- }
|
|
|
|
|
|
+ synchronized (clientCache) {
|
|
|
|
+ // Mark the entry as evicted
|
|
|
|
+ XceiverClientSpi info = removalNotification.getValue();
|
|
|
|
+ info.setEvicted();
|
|
}
|
|
}
|
|
}
|
|
}
|
|
}).build();
|
|
}).build();
|
|
}
|
|
}
|
|
|
|
|
|
|
|
+ @VisibleForTesting
|
|
|
|
+ public Cache<String, XceiverClientSpi> getClientCache() {
|
|
|
|
+ return clientCache;
|
|
|
|
+ }
|
|
|
|
+
|
|
/**
|
|
/**
|
|
- * Acquires a XceiverClient connected to a container capable of storing the
|
|
|
|
- * specified key.
|
|
|
|
|
|
+ * Acquires a SharedXceiverClient connected to a container capable of
|
|
|
|
+ * storing the specified key.
|
|
*
|
|
*
|
|
- * If there is already a cached XceiverClient, simply return the cached
|
|
|
|
- * otherwise create a new one.
|
|
|
|
|
|
+ * If there is already a cached SharedXceiverClient, simply return
|
|
|
|
+ * the cached otherwise create a new one.
|
|
*
|
|
*
|
|
* @param pipeline the container pipeline for the client connection
|
|
* @param pipeline the container pipeline for the client connection
|
|
- * @return XceiverClient connected to a container
|
|
|
|
- * @throws IOException if an XceiverClient cannot be acquired
|
|
|
|
|
|
+ * @return SharedXceiverClient connected to a container
|
|
|
|
+ * @throws IOException if an SharedXceiverClient cannot be acquired
|
|
*/
|
|
*/
|
|
- public XceiverClientSpi acquireClient(Pipeline pipeline) throws IOException {
|
|
|
|
|
|
+ public XceiverClientSpi acquireClient(Pipeline pipeline)
|
|
|
|
+ throws IOException {
|
|
Preconditions.checkNotNull(pipeline);
|
|
Preconditions.checkNotNull(pipeline);
|
|
Preconditions.checkArgument(pipeline.getMachines() != null);
|
|
Preconditions.checkArgument(pipeline.getMachines() != null);
|
|
Preconditions.checkArgument(!pipeline.getMachines().isEmpty());
|
|
Preconditions.checkArgument(!pipeline.getMachines().isEmpty());
|
|
- String containerName = pipeline.getContainerName();
|
|
|
|
- synchronized(openClient) {
|
|
|
|
- XceiverClientWithAccessInfo info =
|
|
|
|
- openClient.getIfPresent(containerName);
|
|
|
|
|
|
|
|
- if (info != null) {
|
|
|
|
- // we do have this connection, add reference and return
|
|
|
|
- info.incrementReference();
|
|
|
|
- return info.getXceiverClient();
|
|
|
|
- } else {
|
|
|
|
- // connection not found, create new, add reference and return
|
|
|
|
- final XceiverClientSpi xceiverClient = useRatis ?
|
|
|
|
- XceiverClientRatis.newXceiverClientRatis(pipeline, conf)
|
|
|
|
- : new XceiverClient(pipeline, conf);
|
|
|
|
- try {
|
|
|
|
- xceiverClient.connect();
|
|
|
|
- } catch (Exception e) {
|
|
|
|
- throw new IOException("Exception connecting XceiverClient.", e);
|
|
|
|
- }
|
|
|
|
- info = new XceiverClientWithAccessInfo(xceiverClient);
|
|
|
|
- info.incrementReference();
|
|
|
|
- openClient.put(containerName, info);
|
|
|
|
- return xceiverClient;
|
|
|
|
- }
|
|
|
|
|
|
+ synchronized (clientCache) {
|
|
|
|
+ XceiverClientSpi info = getClient(pipeline);
|
|
|
|
+ info.incrementReference();
|
|
|
|
+ return info;
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
/**
|
|
/**
|
|
- * Releases an XceiverClient after use.
|
|
|
|
|
|
+ * Releases an SharedXceiverClient after use.
|
|
*
|
|
*
|
|
- * @param xceiverClient client to release
|
|
|
|
|
|
+ * @param client client to release
|
|
*/
|
|
*/
|
|
- public void releaseClient(XceiverClientSpi xceiverClient) {
|
|
|
|
- Preconditions.checkNotNull(xceiverClient);
|
|
|
|
- String containerName = xceiverClient.getPipeline().getContainerName();
|
|
|
|
- XceiverClientWithAccessInfo info;
|
|
|
|
- synchronized (openClient) {
|
|
|
|
- info = openClient.getIfPresent(containerName);
|
|
|
|
- Preconditions.checkNotNull(info);
|
|
|
|
- info.decrementReference();
|
|
|
|
|
|
+ public void releaseClient(XceiverClientSpi client) {
|
|
|
|
+ Preconditions.checkNotNull(client);
|
|
|
|
+ synchronized (clientCache) {
|
|
|
|
+ client.decrementReference();
|
|
}
|
|
}
|
|
}
|
|
}
|
|
|
|
|
|
- /**
|
|
|
|
- * A helper class for caching and cleaning XceiverClient. Three parameters:
|
|
|
|
- * - the actual XceiverClient object
|
|
|
|
- * - a time stamp representing the most recent access (acquire or release)
|
|
|
|
- * - a reference count, +1 when acquire, -1 when release
|
|
|
|
- */
|
|
|
|
- private static class XceiverClientWithAccessInfo {
|
|
|
|
- final private XceiverClientSpi xceiverClient;
|
|
|
|
- final private AtomicInteger referenceCount;
|
|
|
|
-
|
|
|
|
- XceiverClientWithAccessInfo(XceiverClientSpi xceiverClient) {
|
|
|
|
- this.xceiverClient = xceiverClient;
|
|
|
|
- this.referenceCount = new AtomicInteger(0);
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- void incrementReference() {
|
|
|
|
- this.referenceCount.incrementAndGet();
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- void decrementReference() {
|
|
|
|
- this.referenceCount.decrementAndGet();
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- boolean hasRefence() {
|
|
|
|
- return this.referenceCount.get() != 0;
|
|
|
|
- }
|
|
|
|
-
|
|
|
|
- XceiverClientSpi getXceiverClient() {
|
|
|
|
- return xceiverClient;
|
|
|
|
|
|
+ private XceiverClientSpi getClient(Pipeline pipeline)
|
|
|
|
+ throws IOException {
|
|
|
|
+ String containerName = pipeline.getContainerName();
|
|
|
|
+ try {
|
|
|
|
+ return clientCache.get(containerName,
|
|
|
|
+ new Callable<XceiverClientSpi>() {
|
|
|
|
+ @Override
|
|
|
|
+ public XceiverClientSpi call() throws Exception {
|
|
|
|
+ XceiverClientSpi client = useRatis ?
|
|
|
|
+ XceiverClientRatis.newXceiverClientRatis(pipeline, conf)
|
|
|
|
+ : new XceiverClient(pipeline, conf);
|
|
|
|
+ client.connect();
|
|
|
|
+ return client;
|
|
|
|
+ }
|
|
|
|
+ });
|
|
|
|
+ } catch (Exception e) {
|
|
|
|
+ throw new IOException("Exception getting XceiverClient.", e);
|
|
}
|
|
}
|
|
}
|
|
}
|
|
}
|
|
}
|