|
@@ -18,7 +18,6 @@
|
|
|
|
|
|
package org.apache.hadoop.ozone.container.common.transport.server.ratis;
|
|
|
|
|
|
-import com.google.common.annotations.VisibleForTesting;
|
|
|
import com.google.common.base.Preconditions;
|
|
|
import org.apache.hadoop.hdds.HddsUtils;
|
|
|
import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos;
|
|
@@ -26,7 +25,7 @@ import org.apache.hadoop.ozone.container.common.helpers.BlockData;
|
|
|
import org.apache.ratis.protocol.RaftGroup;
|
|
|
import org.apache.ratis.protocol.RaftGroupId;
|
|
|
import org.apache.ratis.server.RaftServer;
|
|
|
-import org.apache.ratis.shaded.com.google.protobuf
|
|
|
+import org.apache.ratis.thirdparty.com.google.protobuf
|
|
|
.InvalidProtocolBufferException;
|
|
|
import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.Type;
|
|
|
import org.apache.hadoop.hdds.protocol.datanode.proto.ContainerProtos.Stage;
|
|
@@ -44,10 +43,10 @@ import org.apache.hadoop.ozone.container.common.interfaces.ContainerDispatcher;
|
|
|
import org.apache.ratis.protocol.Message;
|
|
|
import org.apache.ratis.protocol.RaftClientRequest;
|
|
|
import org.apache.ratis.server.storage.RaftStorage;
|
|
|
-import org.apache.ratis.shaded.com.google.protobuf.ByteString;
|
|
|
-import org.apache.ratis.shaded.proto.RaftProtos.RoleInfoProto;
|
|
|
-import org.apache.ratis.shaded.proto.RaftProtos.LogEntryProto;
|
|
|
-import org.apache.ratis.shaded.proto.RaftProtos.SMLogEntryProto;
|
|
|
+import org.apache.ratis.thirdparty.com.google.protobuf.ByteString;
|
|
|
+import org.apache.ratis.proto.RaftProtos.RoleInfoProto;
|
|
|
+import org.apache.ratis.proto.RaftProtos.LogEntryProto;
|
|
|
+import org.apache.ratis.proto.RaftProtos.SMLogEntryProto;
|
|
|
import org.apache.ratis.statemachine.StateMachineStorage;
|
|
|
import org.apache.ratis.statemachine.TransactionContext;
|
|
|
import org.apache.ratis.statemachine.impl.BaseStateMachine;
|
|
@@ -57,12 +56,12 @@ import org.slf4j.Logger;
|
|
|
import org.slf4j.LoggerFactory;
|
|
|
|
|
|
import java.io.IOException;
|
|
|
-import java.util.ArrayList;
|
|
|
-import java.util.Collection;
|
|
|
import java.util.List;
|
|
|
import java.util.concurrent.CompletableFuture;
|
|
|
import java.util.concurrent.ConcurrentHashMap;
|
|
|
import java.util.concurrent.ThreadPoolExecutor;
|
|
|
+import java.util.concurrent.Executors;
|
|
|
+import java.util.concurrent.ExecutorService;
|
|
|
import java.util.stream.Collectors;
|
|
|
|
|
|
/** A {@link org.apache.ratis.statemachine.StateMachine} for containers.
|
|
@@ -98,44 +97,43 @@ import java.util.stream.Collectors;
|
|
|
*
|
|
|
* 2) Write chunk commit operation is executed after write chunk state machine
|
|
|
* operation. This will ensure that commit operation is sync'd with the state
|
|
|
- * machine operation.
|
|
|
- *
|
|
|
- * Synchronization between {@link #writeStateMachineData} and
|
|
|
- * {@link #applyTransaction} need to be enforced in the StateMachine
|
|
|
- * implementation. For example, synchronization between writeChunk and
|
|
|
+ * machine operation.For example, synchronization between writeChunk and
|
|
|
* createContainer in {@link ContainerStateMachine}.
|
|
|
- *
|
|
|
- * PutBlock is synchronized with WriteChunk operations, PutBlock for a block is
|
|
|
- * executed only after all the WriteChunk preceding the PutBlock have finished.
|
|
|
- *
|
|
|
- * CloseContainer is synchronized with WriteChunk and PutBlock operations,
|
|
|
- * CloseContainer for a container is processed after all the preceding write
|
|
|
- * operations for the container have finished.
|
|
|
- * */
|
|
|
+ **/
|
|
|
+
|
|
|
public class ContainerStateMachine extends BaseStateMachine {
|
|
|
- static final Logger LOG = LoggerFactory.getLogger(
|
|
|
- ContainerStateMachine.class);
|
|
|
- private final SimpleStateMachineStorage storage
|
|
|
- = new SimpleStateMachineStorage();
|
|
|
+ static final Logger LOG =
|
|
|
+ LoggerFactory.getLogger(ContainerStateMachine.class);
|
|
|
+ private final SimpleStateMachineStorage storage =
|
|
|
+ new SimpleStateMachineStorage();
|
|
|
private final ContainerDispatcher dispatcher;
|
|
|
private ThreadPoolExecutor chunkExecutor;
|
|
|
private final XceiverServerRatis ratisServer;
|
|
|
private final ConcurrentHashMap<Long, CompletableFuture<Message>>
|
|
|
writeChunkFutureMap;
|
|
|
- private final ConcurrentHashMap<Long, StateMachineHelper> stateMachineMap;
|
|
|
+ private final ConcurrentHashMap<Long, CompletableFuture<Message>>
|
|
|
+ createContainerFutureMap;
|
|
|
+ private ExecutorService[] executors;
|
|
|
+ private final int numExecutors;
|
|
|
/**
|
|
|
* CSM metrics.
|
|
|
*/
|
|
|
private final CSMMetrics metrics;
|
|
|
|
|
|
public ContainerStateMachine(ContainerDispatcher dispatcher,
|
|
|
- ThreadPoolExecutor chunkExecutor, XceiverServerRatis ratisServer) {
|
|
|
+ ThreadPoolExecutor chunkExecutor, XceiverServerRatis ratisServer,
|
|
|
+ int numOfExecutors) {
|
|
|
this.dispatcher = dispatcher;
|
|
|
this.chunkExecutor = chunkExecutor;
|
|
|
this.ratisServer = ratisServer;
|
|
|
this.writeChunkFutureMap = new ConcurrentHashMap<>();
|
|
|
- this.stateMachineMap = new ConcurrentHashMap<>();
|
|
|
metrics = CSMMetrics.create();
|
|
|
+ this.createContainerFutureMap = new ConcurrentHashMap<>();
|
|
|
+ this.numExecutors = numOfExecutors;
|
|
|
+ executors = new ExecutorService[numExecutors];
|
|
|
+ for (int i = 0; i < numExecutors; i++) {
|
|
|
+ executors[i] = Executors.newSingleThreadExecutor();
|
|
|
+ }
|
|
|
}
|
|
|
|
|
|
@Override
|
|
@@ -229,6 +227,41 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
return dispatchCommand(requestProto)::toByteString;
|
|
|
}
|
|
|
|
|
|
+ private ExecutorService getCommandExecutor(
|
|
|
+ ContainerCommandRequestProto requestProto) {
|
|
|
+ int executorId = (int)(requestProto.getContainerID() % numExecutors);
|
|
|
+ return executors[executorId];
|
|
|
+ }
|
|
|
+
|
|
|
+ private CompletableFuture<Message> handleWriteChunk(
|
|
|
+ ContainerCommandRequestProto requestProto, long entryIndex) {
|
|
|
+ final WriteChunkRequestProto write = requestProto.getWriteChunk();
|
|
|
+ long containerID = write.getBlockID().getContainerID();
|
|
|
+ CompletableFuture<Message> future =
|
|
|
+ createContainerFutureMap.get(containerID);
|
|
|
+ CompletableFuture<Message> writeChunkFuture;
|
|
|
+ if (future != null) {
|
|
|
+ writeChunkFuture = future.thenApplyAsync(
|
|
|
+ v -> runCommand(requestProto), chunkExecutor);
|
|
|
+ } else {
|
|
|
+ writeChunkFuture = CompletableFuture.supplyAsync(
|
|
|
+ () -> runCommand(requestProto), chunkExecutor);
|
|
|
+ }
|
|
|
+ writeChunkFutureMap.put(entryIndex, writeChunkFuture);
|
|
|
+ // Remove the future once it finishes execution from the
|
|
|
+ // writeChunkFutureMap.
|
|
|
+ writeChunkFuture.thenApply(r -> writeChunkFutureMap.remove(entryIndex));
|
|
|
+ return writeChunkFuture;
|
|
|
+ }
|
|
|
+
|
|
|
+ private CompletableFuture<Message> handleCreateContainer(
|
|
|
+ ContainerCommandRequestProto requestProto) {
|
|
|
+ long containerID = requestProto.getContainerID();
|
|
|
+ createContainerFutureMap.
|
|
|
+ computeIfAbsent(containerID, k -> new CompletableFuture<>());
|
|
|
+ return CompletableFuture.completedFuture(() -> ByteString.EMPTY);
|
|
|
+ }
|
|
|
+
|
|
|
/*
|
|
|
* writeStateMachineData calls are not synchronized with each other
|
|
|
* and also with applyTransaction.
|
|
@@ -240,17 +273,15 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
final ContainerCommandRequestProto requestProto =
|
|
|
getRequestProto(entry.getSmLogEntry().getStateMachineData());
|
|
|
Type cmdType = requestProto.getCmdType();
|
|
|
- long containerId = requestProto.getContainerID();
|
|
|
- stateMachineMap
|
|
|
- .computeIfAbsent(containerId, k -> new StateMachineHelper());
|
|
|
- CompletableFuture<Message> stateMachineFuture =
|
|
|
- stateMachineMap.get(containerId)
|
|
|
- .handleStateMachineData(requestProto, entry.getIndex());
|
|
|
- if (stateMachineFuture == null) {
|
|
|
- throw new IllegalStateException(
|
|
|
- "Cmd Type:" + cmdType + " should not have state machine data");
|
|
|
+ switch (cmdType) {
|
|
|
+ case CreateContainer:
|
|
|
+ return handleCreateContainer(requestProto);
|
|
|
+ case WriteChunk:
|
|
|
+ return handleWriteChunk(requestProto, entry.getIndex());
|
|
|
+ default:
|
|
|
+ throw new IllegalStateException("Cmd Type:" + cmdType
|
|
|
+ + " should not have state machine data");
|
|
|
}
|
|
|
- return stateMachineFuture;
|
|
|
} catch (IOException e) {
|
|
|
metrics.incNumWriteStateMachineFails();
|
|
|
return completeExceptionally(e);
|
|
@@ -270,14 +301,14 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
}
|
|
|
}
|
|
|
|
|
|
- private LogEntryProto readStateMachineData(LogEntryProto entry,
|
|
|
+ private ByteString readStateMachineData(LogEntryProto entry,
|
|
|
ContainerCommandRequestProto requestProto) {
|
|
|
WriteChunkRequestProto writeChunkRequestProto =
|
|
|
requestProto.getWriteChunk();
|
|
|
// Assert that store log entry is for COMMIT_DATA, the WRITE_DATA is
|
|
|
// written through writeStateMachineData.
|
|
|
- Preconditions.checkArgument(writeChunkRequestProto.getStage()
|
|
|
- == Stage.COMMIT_DATA);
|
|
|
+ Preconditions
|
|
|
+ .checkArgument(writeChunkRequestProto.getStage() == Stage.COMMIT_DATA);
|
|
|
|
|
|
// prepare the chunk to be read
|
|
|
ReadChunkRequestProto.Builder readChunkRequestProto =
|
|
@@ -286,8 +317,7 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
.setChunkData(writeChunkRequestProto.getChunkData());
|
|
|
ContainerCommandRequestProto dataContainerCommandProto =
|
|
|
ContainerCommandRequestProto.newBuilder(requestProto)
|
|
|
- .setCmdType(Type.ReadChunk)
|
|
|
- .setReadChunk(readChunkRequestProto)
|
|
|
+ .setCmdType(Type.ReadChunk).setReadChunk(readChunkRequestProto)
|
|
|
.build();
|
|
|
|
|
|
// read the chunk
|
|
@@ -302,25 +332,13 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
final WriteChunkRequestProto.Builder dataWriteChunkProto =
|
|
|
WriteChunkRequestProto.newBuilder(writeChunkRequestProto)
|
|
|
// adding the state machine data
|
|
|
- .setData(responseProto.getData())
|
|
|
- .setStage(Stage.WRITE_DATA);
|
|
|
+ .setData(responseProto.getData()).setStage(Stage.WRITE_DATA);
|
|
|
|
|
|
ContainerCommandRequestProto.Builder newStateMachineProto =
|
|
|
ContainerCommandRequestProto.newBuilder(requestProto)
|
|
|
.setWriteChunk(dataWriteChunkProto);
|
|
|
|
|
|
- return recreateLogEntryProto(entry,
|
|
|
- newStateMachineProto.build().toByteString());
|
|
|
- }
|
|
|
-
|
|
|
- private LogEntryProto recreateLogEntryProto(LogEntryProto entry,
|
|
|
- ByteString stateMachineData) {
|
|
|
- // recreate the log entry
|
|
|
- final SMLogEntryProto log =
|
|
|
- SMLogEntryProto.newBuilder(entry.getSmLogEntry())
|
|
|
- .setStateMachineData(stateMachineData)
|
|
|
- .build();
|
|
|
- return LogEntryProto.newBuilder(entry).setSmLogEntry(log).build();
|
|
|
+ return newStateMachineProto.build().toByteString();
|
|
|
}
|
|
|
|
|
|
/**
|
|
@@ -347,11 +365,11 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
* evicted.
|
|
|
*/
|
|
|
@Override
|
|
|
- public CompletableFuture<LogEntryProto> readStateMachineData(
|
|
|
+ public CompletableFuture<ByteString> readStateMachineData(
|
|
|
LogEntryProto entry) {
|
|
|
SMLogEntryProto smLogEntryProto = entry.getSmLogEntry();
|
|
|
if (!smLogEntryProto.getStateMachineData().isEmpty()) {
|
|
|
- return CompletableFuture.completedFuture(entry);
|
|
|
+ return CompletableFuture.completedFuture(ByteString.EMPTY);
|
|
|
}
|
|
|
|
|
|
try {
|
|
@@ -365,9 +383,7 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
readStateMachineData(entry, requestProto),
|
|
|
chunkExecutor);
|
|
|
} else if (requestProto.getCmdType() == Type.CreateContainer) {
|
|
|
- LogEntryProto log =
|
|
|
- recreateLogEntryProto(entry, requestProto.toByteString());
|
|
|
- return CompletableFuture.completedFuture(log);
|
|
|
+ return CompletableFuture.completedFuture(requestProto.toByteString());
|
|
|
} else {
|
|
|
throw new IllegalStateException("Cmd type:" + requestProto.getCmdType()
|
|
|
+ " cannot have state machine data");
|
|
@@ -387,13 +403,44 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
metrics.incNumApplyTransactionsOps();
|
|
|
ContainerCommandRequestProto requestProto =
|
|
|
getRequestProto(trx.getSMLogEntry().getData());
|
|
|
- Preconditions.checkState(!HddsUtils.isReadOnly(requestProto));
|
|
|
- stateMachineMap.computeIfAbsent(requestProto.getContainerID(),
|
|
|
- k -> new StateMachineHelper());
|
|
|
- long index =
|
|
|
- trx.getLogEntry() == null ? -1 : trx.getLogEntry().getIndex();
|
|
|
- return stateMachineMap.get(requestProto.getContainerID())
|
|
|
- .executeContainerCommand(requestProto, index);
|
|
|
+ Type cmdType = requestProto.getCmdType();
|
|
|
+ CompletableFuture<Message> future;
|
|
|
+ if (cmdType == Type.PutBlock) {
|
|
|
+ BlockData blockData;
|
|
|
+ ContainerProtos.BlockData blockDataProto =
|
|
|
+ requestProto.getPutBlock().getBlockData();
|
|
|
+
|
|
|
+ // set the blockCommitSequenceId
|
|
|
+ try {
|
|
|
+ blockData = BlockData.getFromProtoBuf(blockDataProto);
|
|
|
+ } catch (IOException ioe) {
|
|
|
+ LOG.error("unable to retrieve blockData info for Block {}",
|
|
|
+ blockDataProto.getBlockID());
|
|
|
+ return completeExceptionally(ioe);
|
|
|
+ }
|
|
|
+ blockData.setBlockCommitSequenceId(trx.getLogEntry().getIndex());
|
|
|
+ final ContainerProtos.PutBlockRequestProto putBlockRequestProto =
|
|
|
+ ContainerProtos.PutBlockRequestProto
|
|
|
+ .newBuilder(requestProto.getPutBlock())
|
|
|
+ .setBlockData(blockData.getProtoBufMessage()).build();
|
|
|
+ ContainerCommandRequestProto containerCommandRequestProto =
|
|
|
+ ContainerCommandRequestProto.newBuilder(requestProto)
|
|
|
+ .setPutBlock(putBlockRequestProto).build();
|
|
|
+ future = CompletableFuture
|
|
|
+ .supplyAsync(() -> runCommand(containerCommandRequestProto),
|
|
|
+ getCommandExecutor(requestProto));
|
|
|
+ } else {
|
|
|
+ future = CompletableFuture.supplyAsync(() -> runCommand(requestProto),
|
|
|
+ getCommandExecutor(requestProto));
|
|
|
+ }
|
|
|
+ // Mark the createContainerFuture complete so that writeStateMachineData
|
|
|
+ // for WriteChunk gets unblocked
|
|
|
+ if (cmdType == Type.CreateContainer) {
|
|
|
+ long containerID = requestProto.getContainerID();
|
|
|
+ future.thenApply(
|
|
|
+ r -> createContainerFutureMap.remove(containerID).complete(null));
|
|
|
+ }
|
|
|
+ return future;
|
|
|
} catch (IOException e) {
|
|
|
metrics.incNumApplyTransactionsFails();
|
|
|
return completeExceptionally(e);
|
|
@@ -419,259 +466,8 @@ public class ContainerStateMachine extends BaseStateMachine {
|
|
|
|
|
|
@Override
|
|
|
public void close() throws IOException {
|
|
|
- }
|
|
|
-
|
|
|
- /**
|
|
|
- * Class to manage the future tasks for writeChunks.
|
|
|
- */
|
|
|
- static class CommitChunkFutureMap {
|
|
|
- private final ConcurrentHashMap<Long, CompletableFuture<Message>>
|
|
|
- block2ChunkMap = new ConcurrentHashMap<>();
|
|
|
-
|
|
|
- synchronized int removeAndGetSize(long index) {
|
|
|
- block2ChunkMap.remove(index);
|
|
|
- return block2ChunkMap.size();
|
|
|
+ for (int i = 0; i < numExecutors; i++){
|
|
|
+ executors[i].shutdown();
|
|
|
}
|
|
|
-
|
|
|
- synchronized CompletableFuture<Message> add(long index,
|
|
|
- CompletableFuture<Message> future) {
|
|
|
- return block2ChunkMap.put(index, future);
|
|
|
- }
|
|
|
-
|
|
|
- synchronized List<CompletableFuture<Message>> getAll() {
|
|
|
- return new ArrayList<>(block2ChunkMap.values());
|
|
|
- }
|
|
|
- }
|
|
|
-
|
|
|
- /**
|
|
|
- * This class maintains maps and provide utilities to enforce synchronization
|
|
|
- * among createContainer, writeChunk, putBlock and closeContainer.
|
|
|
- */
|
|
|
- private class StateMachineHelper {
|
|
|
-
|
|
|
- private CompletableFuture<Message> createContainerFuture;
|
|
|
-
|
|
|
- // Map for maintaining all writeChunk futures mapped to blockId
|
|
|
- private final ConcurrentHashMap<Long, CommitChunkFutureMap>
|
|
|
- block2ChunkMap;
|
|
|
-
|
|
|
- // Map for putBlock futures
|
|
|
- private final ConcurrentHashMap<Long, CompletableFuture<Message>>
|
|
|
- blockCommitMap;
|
|
|
-
|
|
|
- StateMachineHelper() {
|
|
|
- createContainerFuture = null;
|
|
|
- block2ChunkMap = new ConcurrentHashMap<>();
|
|
|
- blockCommitMap = new ConcurrentHashMap<>();
|
|
|
- }
|
|
|
-
|
|
|
- // The following section handles writeStateMachineData transactions
|
|
|
- // on a container
|
|
|
-
|
|
|
- // enqueue the create container future during writeStateMachineData
|
|
|
- // so that the write stateMachine data phase of writeChunk wait on
|
|
|
- // create container to finish.
|
|
|
- private CompletableFuture<Message> handleCreateContainer() {
|
|
|
- createContainerFuture = new CompletableFuture<>();
|
|
|
- return CompletableFuture.completedFuture(() -> ByteString.EMPTY);
|
|
|
- }
|
|
|
-
|
|
|
- // This synchronizes on create container to finish
|
|
|
- private CompletableFuture<Message> handleWriteChunk(
|
|
|
- ContainerCommandRequestProto requestProto, long entryIndex) {
|
|
|
- CompletableFuture<Message> containerOpFuture;
|
|
|
-
|
|
|
- if (createContainerFuture != null) {
|
|
|
- containerOpFuture = createContainerFuture
|
|
|
- .thenApplyAsync(v -> runCommand(requestProto), chunkExecutor);
|
|
|
- } else {
|
|
|
- containerOpFuture = CompletableFuture
|
|
|
- .supplyAsync(() -> runCommand(requestProto), chunkExecutor);
|
|
|
- }
|
|
|
- writeChunkFutureMap.put(entryIndex, containerOpFuture);
|
|
|
- return containerOpFuture;
|
|
|
- }
|
|
|
-
|
|
|
- CompletableFuture<Message> handleStateMachineData(
|
|
|
- final ContainerCommandRequestProto requestProto, long index) {
|
|
|
- Type cmdType = requestProto.getCmdType();
|
|
|
- if (cmdType == Type.CreateContainer) {
|
|
|
- return handleCreateContainer();
|
|
|
- } else if (cmdType == Type.WriteChunk) {
|
|
|
- return handleWriteChunk(requestProto, index);
|
|
|
- } else {
|
|
|
- return null;
|
|
|
- }
|
|
|
- }
|
|
|
-
|
|
|
- // The following section handles applyTransaction transactions
|
|
|
- // on a container
|
|
|
-
|
|
|
- private CompletableFuture<Message> handlePutBlock(
|
|
|
- ContainerCommandRequestProto requestProto, long index) {
|
|
|
- List<CompletableFuture<Message>> futureList = new ArrayList<>();
|
|
|
- BlockData blockData = null;
|
|
|
- ContainerProtos.BlockData blockDataProto =
|
|
|
- requestProto.getPutBlock().getBlockData();
|
|
|
-
|
|
|
- // set the blockCommitSequenceId
|
|
|
- try {
|
|
|
- blockData = BlockData.getFromProtoBuf(blockDataProto);
|
|
|
- } catch (IOException ioe) {
|
|
|
- LOG.error("unable to retrieve blockData info for Block {}",
|
|
|
- blockDataProto.getBlockID());
|
|
|
- return completeExceptionally(ioe);
|
|
|
- }
|
|
|
- blockData.setBlockCommitSequenceId(index);
|
|
|
- final ContainerProtos.PutBlockRequestProto putBlockRequestProto =
|
|
|
- ContainerProtos.PutBlockRequestProto
|
|
|
- .newBuilder(requestProto.getPutBlock())
|
|
|
- .setBlockData(blockData.getProtoBufMessage()).build();
|
|
|
- ContainerCommandRequestProto containerCommandRequestProto =
|
|
|
- ContainerCommandRequestProto.newBuilder(requestProto)
|
|
|
- .setPutBlock(putBlockRequestProto).build();
|
|
|
- long localId = blockDataProto.getBlockID().getLocalID();
|
|
|
- // Need not wait for create container future here as it has already
|
|
|
- // finished.
|
|
|
- if (block2ChunkMap.get(localId) != null) {
|
|
|
- futureList.addAll(block2ChunkMap.get(localId).getAll());
|
|
|
- }
|
|
|
- CompletableFuture<Message> effectiveFuture =
|
|
|
- runCommandAfterFutures(futureList, containerCommandRequestProto);
|
|
|
-
|
|
|
- CompletableFuture<Message> putBlockFuture =
|
|
|
- effectiveFuture.thenApply(message -> {
|
|
|
- blockCommitMap.remove(localId);
|
|
|
- return message;
|
|
|
- });
|
|
|
- blockCommitMap.put(localId, putBlockFuture);
|
|
|
- return putBlockFuture;
|
|
|
- }
|
|
|
-
|
|
|
- // Close Container should be executed only if all pending WriteType
|
|
|
- // container cmds get executed. Transactions which can return a future
|
|
|
- // are WriteChunk and PutBlock.
|
|
|
- private CompletableFuture<Message> handleCloseContainer(
|
|
|
- ContainerCommandRequestProto requestProto) {
|
|
|
- List<CompletableFuture<Message>> futureList = new ArrayList<>();
|
|
|
-
|
|
|
- // No need to wait for create container future here as it should have
|
|
|
- // already finished.
|
|
|
- block2ChunkMap.values().forEach(b -> futureList.addAll(b.getAll()));
|
|
|
- futureList.addAll(blockCommitMap.values());
|
|
|
-
|
|
|
- // There are pending write Chunk/PutBlock type requests
|
|
|
- // Queue this closeContainer request behind all these requests
|
|
|
- CompletableFuture<Message> closeContainerFuture =
|
|
|
- runCommandAfterFutures(futureList, requestProto);
|
|
|
-
|
|
|
- return closeContainerFuture.thenApply(message -> {
|
|
|
- stateMachineMap.remove(requestProto.getContainerID());
|
|
|
- return message;
|
|
|
- });
|
|
|
- }
|
|
|
-
|
|
|
- private CompletableFuture<Message> handleChunkCommit(
|
|
|
- ContainerCommandRequestProto requestProto, long index) {
|
|
|
- WriteChunkRequestProto write = requestProto.getWriteChunk();
|
|
|
- // the data field has already been removed in start Transaction
|
|
|
- Preconditions.checkArgument(!write.hasData());
|
|
|
- CompletableFuture<Message> stateMachineFuture =
|
|
|
- writeChunkFutureMap.remove(index);
|
|
|
- CompletableFuture<Message> commitChunkFuture = stateMachineFuture
|
|
|
- .thenComposeAsync(v -> CompletableFuture
|
|
|
- .completedFuture(runCommand(requestProto)));
|
|
|
-
|
|
|
- long localId = requestProto.getWriteChunk().getBlockID().getLocalID();
|
|
|
- // Put the applyTransaction Future again to the Map.
|
|
|
- // closeContainer should synchronize with this.
|
|
|
- block2ChunkMap
|
|
|
- .computeIfAbsent(localId, id -> new CommitChunkFutureMap())
|
|
|
- .add(index, commitChunkFuture);
|
|
|
- return commitChunkFuture.thenApply(message -> {
|
|
|
- block2ChunkMap.computeIfPresent(localId, (containerId, chunks)
|
|
|
- -> chunks.removeAndGetSize(index) == 0? null: chunks);
|
|
|
- return message;
|
|
|
- });
|
|
|
- }
|
|
|
-
|
|
|
- private CompletableFuture<Message> runCommandAfterFutures(
|
|
|
- List<CompletableFuture<Message>> futureList,
|
|
|
- ContainerCommandRequestProto requestProto) {
|
|
|
- CompletableFuture<Message> effectiveFuture;
|
|
|
- if (futureList.isEmpty()) {
|
|
|
- effectiveFuture = CompletableFuture
|
|
|
- .supplyAsync(() -> runCommand(requestProto));
|
|
|
-
|
|
|
- } else {
|
|
|
- CompletableFuture<Void> allFuture = CompletableFuture.allOf(
|
|
|
- futureList.toArray(new CompletableFuture[futureList.size()]));
|
|
|
- effectiveFuture = allFuture
|
|
|
- .thenApplyAsync(v -> runCommand(requestProto));
|
|
|
- }
|
|
|
- return effectiveFuture;
|
|
|
- }
|
|
|
-
|
|
|
- CompletableFuture<Message> handleCreateContainer(
|
|
|
- ContainerCommandRequestProto requestProto) {
|
|
|
- CompletableFuture<Message> future =
|
|
|
- CompletableFuture.completedFuture(runCommand(requestProto));
|
|
|
- future.thenAccept(m -> {
|
|
|
- createContainerFuture.complete(m);
|
|
|
- createContainerFuture = null;
|
|
|
- });
|
|
|
- return future;
|
|
|
- }
|
|
|
-
|
|
|
- CompletableFuture<Message> handleOtherCommands(
|
|
|
- ContainerCommandRequestProto requestProto) {
|
|
|
- return CompletableFuture.completedFuture(runCommand(requestProto));
|
|
|
- }
|
|
|
-
|
|
|
- CompletableFuture<Message> executeContainerCommand(
|
|
|
- ContainerCommandRequestProto requestProto, long index) {
|
|
|
- Type cmdType = requestProto.getCmdType();
|
|
|
- switch (cmdType) {
|
|
|
- case WriteChunk:
|
|
|
- return handleChunkCommit(requestProto, index);
|
|
|
- case CloseContainer:
|
|
|
- return handleCloseContainer(requestProto);
|
|
|
- case PutBlock:
|
|
|
- return handlePutBlock(requestProto, index);
|
|
|
- case CreateContainer:
|
|
|
- return handleCreateContainer(requestProto);
|
|
|
- default:
|
|
|
- return handleOtherCommands(requestProto);
|
|
|
- }
|
|
|
- }
|
|
|
- }
|
|
|
-
|
|
|
- @VisibleForTesting
|
|
|
- public ConcurrentHashMap<Long, StateMachineHelper> getStateMachineMap() {
|
|
|
- return stateMachineMap;
|
|
|
- }
|
|
|
-
|
|
|
- @VisibleForTesting
|
|
|
- public CompletableFuture<Message> getCreateContainerFuture(long containerId) {
|
|
|
- StateMachineHelper helper = stateMachineMap.get(containerId);
|
|
|
- return helper == null ? null : helper.createContainerFuture;
|
|
|
- }
|
|
|
-
|
|
|
- @VisibleForTesting
|
|
|
- public List<CompletableFuture<Message>> getCommitChunkFutureMap(
|
|
|
- long containerId) {
|
|
|
- StateMachineHelper helper = stateMachineMap.get(containerId);
|
|
|
- if (helper != null) {
|
|
|
- List<CompletableFuture<Message>> futureList = new ArrayList<>();
|
|
|
- stateMachineMap.get(containerId).block2ChunkMap.values()
|
|
|
- .forEach(b -> futureList.addAll(b.getAll()));
|
|
|
- return futureList;
|
|
|
- }
|
|
|
- return null;
|
|
|
- }
|
|
|
-
|
|
|
- @VisibleForTesting
|
|
|
- public Collection<CompletableFuture<Message>> getWriteChunkFutureMap() {
|
|
|
- return writeChunkFutureMap.values();
|
|
|
}
|
|
|
}
|