|
@@ -24,6 +24,7 @@ import org.apache.hadoop.hdds.protocol.DatanodeDetails;
|
|
|
import org.apache.hadoop.hdds.protocol.proto.HddsProtos;
|
|
|
import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationFactor;
|
|
|
import org.apache.hadoop.hdds.protocol.proto.HddsProtos.ReplicationType;
|
|
|
+import org.apache.hadoop.hdds.scm.ScmConfigKeys;
|
|
|
import org.apache.hadoop.hdds.scm.container.ContainerID;
|
|
|
import org.apache.hadoop.hdds.scm.events.SCMEvents;
|
|
|
import org.apache.hadoop.hdds.scm.node.NodeManager;
|
|
@@ -34,6 +35,7 @@ import org.apache.hadoop.ozone.OzoneConsts;
|
|
|
import org.apache.hadoop.utils.MetadataKeyFilters;
|
|
|
import org.apache.hadoop.utils.MetadataStore;
|
|
|
import org.apache.hadoop.utils.MetadataStoreBuilder;
|
|
|
+import org.apache.hadoop.utils.Scheduler;
|
|
|
import org.slf4j.Logger;
|
|
|
import org.slf4j.LoggerFactory;
|
|
|
|
|
@@ -46,6 +48,7 @@ import java.util.Map;
|
|
|
import java.util.NavigableSet;
|
|
|
import java.util.Set;
|
|
|
import java.util.Collection;
|
|
|
+import java.util.concurrent.TimeUnit;
|
|
|
import java.util.concurrent.locks.ReadWriteLock;
|
|
|
import java.util.concurrent.locks.ReentrantReadWriteLock;
|
|
|
|
|
@@ -68,19 +71,27 @@ public class SCMPipelineManager implements PipelineManager {
|
|
|
private final ReadWriteLock lock;
|
|
|
private final PipelineFactory pipelineFactory;
|
|
|
private final PipelineStateManager stateManager;
|
|
|
- private final MetadataStore pipelineStore;
|
|
|
+ private final BackgroundPipelineCreator backgroundPipelineCreator;
|
|
|
+ private Scheduler scheduler;
|
|
|
+ private MetadataStore pipelineStore;
|
|
|
|
|
|
private final EventPublisher eventPublisher;
|
|
|
private final NodeManager nodeManager;
|
|
|
private final SCMPipelineMetrics metrics;
|
|
|
+ private final Configuration conf;
|
|
|
// Pipeline Manager MXBean
|
|
|
private ObjectName pmInfoBean;
|
|
|
|
|
|
public SCMPipelineManager(Configuration conf, NodeManager nodeManager,
|
|
|
EventPublisher eventPublisher) throws IOException {
|
|
|
this.lock = new ReentrantReadWriteLock();
|
|
|
+ this.conf = conf;
|
|
|
this.stateManager = new PipelineStateManager(conf);
|
|
|
this.pipelineFactory = new PipelineFactory(nodeManager, stateManager, conf);
|
|
|
+ // TODO: See if thread priority needs to be set for these threads
|
|
|
+ scheduler = new Scheduler("RatisPipelineUtilsThread", false, 1);
|
|
|
+ this.backgroundPipelineCreator =
|
|
|
+ new BackgroundPipelineCreator(this, scheduler, conf);
|
|
|
int cacheSize = conf.getInt(OZONE_SCM_DB_CACHE_SIZE_MB,
|
|
|
OZONE_SCM_DB_CACHE_SIZE_DEFAULT);
|
|
|
final File metaDir = ServerUtils.getScmDbDir(conf);
|
|
@@ -268,35 +279,116 @@ public class SCMPipelineManager implements PipelineManager {
|
|
|
}
|
|
|
|
|
|
@Override
|
|
|
- public void finalizePipeline(PipelineID pipelineId) throws IOException {
|
|
|
+ public void openPipeline(PipelineID pipelineId) throws IOException {
|
|
|
lock.writeLock().lock();
|
|
|
try {
|
|
|
- stateManager.finalizePipeline(pipelineId);
|
|
|
- Set<ContainerID> containerIDs = stateManager.getContainers(pipelineId);
|
|
|
- for (ContainerID containerID : containerIDs) {
|
|
|
- eventPublisher.fireEvent(SCMEvents.CLOSE_CONTAINER, containerID);
|
|
|
- }
|
|
|
+ stateManager.openPipeline(pipelineId);
|
|
|
} finally {
|
|
|
lock.writeLock().unlock();
|
|
|
}
|
|
|
}
|
|
|
|
|
|
+ /**
|
|
|
+ * Finalizes pipeline in the SCM. Removes pipeline and makes rpc call to
|
|
|
+ * destroy pipeline on the datanodes immediately or after timeout based on the
|
|
|
+ * value of onTimeout parameter.
|
|
|
+ *
|
|
|
+ * @param pipeline - Pipeline to be destroyed
|
|
|
+ * @param onTimeout - if true pipeline is removed and destroyed on
|
|
|
+ * datanodes after timeout
|
|
|
+ * @throws IOException
|
|
|
+ */
|
|
|
@Override
|
|
|
- public void openPipeline(PipelineID pipelineId) throws IOException {
|
|
|
+ public void finalizeAndDestroyPipeline(Pipeline pipeline, boolean onTimeout)
|
|
|
+ throws IOException {
|
|
|
+ LOG.info("destroying pipeline:{}", pipeline);
|
|
|
+ finalizePipeline(pipeline.getId());
|
|
|
+ if (onTimeout) {
|
|
|
+ long pipelineDestroyTimeoutInMillis =
|
|
|
+ conf.getTimeDuration(ScmConfigKeys.OZONE_SCM_PIPELINE_DESTROY_TIMEOUT,
|
|
|
+ ScmConfigKeys.OZONE_SCM_PIPELINE_DESTROY_TIMEOUT_DEFAULT,
|
|
|
+ TimeUnit.MILLISECONDS);
|
|
|
+ scheduler.schedule(() -> destroyPipeline(pipeline),
|
|
|
+ pipelineDestroyTimeoutInMillis, TimeUnit.MILLISECONDS, LOG,
|
|
|
+ String.format("Destroy pipeline failed for pipeline:%s", pipeline));
|
|
|
+ } else {
|
|
|
+ destroyPipeline(pipeline);
|
|
|
+ }
|
|
|
+ }
|
|
|
+
|
|
|
+ @Override
|
|
|
+ public Map<String, Integer> getPipelineInfo() {
|
|
|
+ final Map<String, Integer> pipelineInfo = new HashMap<>();
|
|
|
+ for (Pipeline.PipelineState state : Pipeline.PipelineState.values()) {
|
|
|
+ pipelineInfo.put(state.toString(), 0);
|
|
|
+ }
|
|
|
+ stateManager.getPipelines().forEach(pipeline ->
|
|
|
+ pipelineInfo.computeIfPresent(
|
|
|
+ pipeline.getPipelineState().toString(), (k, v) -> v + 1));
|
|
|
+ return pipelineInfo;
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Schedules a fixed interval job to create pipelines.
|
|
|
+ */
|
|
|
+ @Override
|
|
|
+ public void startPipelineCreator() {
|
|
|
+ backgroundPipelineCreator.startFixedIntervalPipelineCreator();
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Triggers pipeline creation after the specified time.
|
|
|
+ */
|
|
|
+ @Override
|
|
|
+ public void triggerPipelineCreation() {
|
|
|
+ backgroundPipelineCreator.triggerPipelineCreation();
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Moves the pipeline to CLOSED state and sends close container command for
|
|
|
+ * all the containers in the pipeline.
|
|
|
+ *
|
|
|
+ * @param pipelineId - ID of the pipeline to be moved to CLOSED state.
|
|
|
+ * @throws IOException
|
|
|
+ */
|
|
|
+ private void finalizePipeline(PipelineID pipelineId) throws IOException {
|
|
|
lock.writeLock().lock();
|
|
|
try {
|
|
|
- stateManager.openPipeline(pipelineId);
|
|
|
+ stateManager.finalizePipeline(pipelineId);
|
|
|
+ Set<ContainerID> containerIDs = stateManager.getContainers(pipelineId);
|
|
|
+ for (ContainerID containerID : containerIDs) {
|
|
|
+ eventPublisher.fireEvent(SCMEvents.CLOSE_CONTAINER, containerID);
|
|
|
+ }
|
|
|
} finally {
|
|
|
lock.writeLock().unlock();
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- @Override
|
|
|
- public void removePipeline(PipelineID pipelineID) throws IOException {
|
|
|
+ /**
|
|
|
+ * Removes pipeline from SCM. Sends ratis command to destroy pipeline on all
|
|
|
+ * the datanodes for ratis pipelines.
|
|
|
+ *
|
|
|
+ * @param pipeline - Pipeline to be destroyed
|
|
|
+ * @throws IOException
|
|
|
+ */
|
|
|
+ private void destroyPipeline(Pipeline pipeline) throws IOException {
|
|
|
+ RatisPipelineUtils.destroyPipeline(pipeline, conf);
|
|
|
+ // remove the pipeline from the pipeline manager
|
|
|
+ removePipeline(pipeline.getId());
|
|
|
+ triggerPipelineCreation();
|
|
|
+ }
|
|
|
+
|
|
|
+ /**
|
|
|
+ * Removes the pipeline from the db and pipeline state map.
|
|
|
+ *
|
|
|
+ * @param pipelineId - ID of the pipeline to be removed
|
|
|
+ * @throws IOException
|
|
|
+ */
|
|
|
+ private void removePipeline(PipelineID pipelineId) throws IOException {
|
|
|
lock.writeLock().lock();
|
|
|
try {
|
|
|
- pipelineStore.delete(pipelineID.getProtobuf().toByteArray());
|
|
|
- Pipeline pipeline = stateManager.removePipeline(pipelineID);
|
|
|
+ pipelineStore.delete(pipelineId.getProtobuf().toByteArray());
|
|
|
+ Pipeline pipeline = stateManager.removePipeline(pipelineId);
|
|
|
nodeManager.removePipeline(pipeline);
|
|
|
metrics.incNumPipelineDestroyed();
|
|
|
} catch (IOException ex) {
|
|
@@ -307,26 +399,16 @@ public class SCMPipelineManager implements PipelineManager {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- @Override
|
|
|
- public Map<String, Integer> getPipelineInfo() {
|
|
|
- final Map<String, Integer> pipelineInfo = new HashMap<>();
|
|
|
- for (Pipeline.PipelineState state : Pipeline.PipelineState.values()) {
|
|
|
- pipelineInfo.put(state.toString(), 0);
|
|
|
- }
|
|
|
- stateManager.getPipelines().forEach(pipeline ->
|
|
|
- pipelineInfo.computeIfPresent(
|
|
|
- pipeline.getPipelineState().toString(), (k, v) -> v + 1));
|
|
|
- return pipelineInfo;
|
|
|
- }
|
|
|
-
|
|
|
@Override
|
|
|
public void close() throws IOException {
|
|
|
- if (pipelineFactory != null) {
|
|
|
- pipelineFactory.close();
|
|
|
+ if (scheduler != null) {
|
|
|
+ scheduler.close();
|
|
|
+ scheduler = null;
|
|
|
}
|
|
|
|
|
|
if (pipelineStore != null) {
|
|
|
pipelineStore.close();
|
|
|
+ pipelineStore = null;
|
|
|
}
|
|
|
if(pmInfoBean != null) {
|
|
|
MBeans.unregister(this.pmInfoBean);
|