X-Git-Url: https://git.opendaylight.org/gerrit/gitweb?p=controller.git;a=blobdiff_plain;f=opendaylight%2Fmd-sal%2Fsal-distributed-datastore%2Fsrc%2Fmain%2Fjava%2Forg%2Fopendaylight%2Fcontroller%2Fcluster%2Fdatastore%2FShard.java;h=ebe5498481d9fe3befdabafbb4976f635a7b15b3;hp=c98e11d60a2e063395ed0b428d2eb5baaee77ea2;hb=d7c9a8ccfcb57f005490a226803d094289997ef9;hpb=057b787289f7b909d7013c22ac73a1c91c860af8 diff --git a/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java b/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java index c98e11d60a..ebe5498481 100644 --- a/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java +++ b/opendaylight/md-sal/sal-distributed-datastore/src/main/java/org/opendaylight/controller/cluster/datastore/Shard.java @@ -11,20 +11,50 @@ package org.opendaylight.controller.cluster.datastore; import akka.actor.ActorRef; import akka.actor.ActorSelection; import akka.actor.Cancellable; +import akka.actor.ExtendedActorSystem; import akka.actor.Props; +import akka.actor.Status; +import akka.actor.Status.Failure; +import akka.serialization.JavaSerializer; import akka.serialization.Serialization; import com.google.common.annotations.VisibleForTesting; import com.google.common.base.Optional; import com.google.common.base.Preconditions; import com.google.common.base.Ticker; +import com.google.common.base.Verify; +import com.google.common.collect.ImmutableList; +import com.google.common.collect.ImmutableMap; +import com.google.common.collect.Range; import java.io.IOException; +import java.util.Arrays; import java.util.Collection; import java.util.Collections; import java.util.Map; import java.util.concurrent.TimeUnit; import javax.annotation.Nonnull; +import javax.annotation.Nullable; +import org.opendaylight.controller.cluster.access.ABIVersion; +import org.opendaylight.controller.cluster.access.commands.ConnectClientRequest; +import org.opendaylight.controller.cluster.access.commands.ConnectClientSuccess; +import org.opendaylight.controller.cluster.access.commands.LocalHistoryRequest; +import org.opendaylight.controller.cluster.access.commands.NotLeaderException; +import org.opendaylight.controller.cluster.access.commands.OutOfSequenceEnvelopeException; +import org.opendaylight.controller.cluster.access.commands.TransactionRequest; +import org.opendaylight.controller.cluster.access.concepts.ClientIdentifier; +import org.opendaylight.controller.cluster.access.concepts.FrontendIdentifier; +import org.opendaylight.controller.cluster.access.concepts.LocalHistoryIdentifier; +import org.opendaylight.controller.cluster.access.concepts.Request; +import org.opendaylight.controller.cluster.access.concepts.RequestEnvelope; +import org.opendaylight.controller.cluster.access.concepts.RequestException; +import org.opendaylight.controller.cluster.access.concepts.RequestSuccess; +import org.opendaylight.controller.cluster.access.concepts.RetiredGenerationException; +import org.opendaylight.controller.cluster.access.concepts.RuntimeRequestException; +import org.opendaylight.controller.cluster.access.concepts.SliceableMessage; import org.opendaylight.controller.cluster.access.concepts.TransactionIdentifier; +import org.opendaylight.controller.cluster.access.concepts.UnsupportedRequestException; import org.opendaylight.controller.cluster.common.actor.CommonConfig; +import org.opendaylight.controller.cluster.common.actor.Dispatchers; +import org.opendaylight.controller.cluster.common.actor.Dispatchers.DispatcherType; import org.opendaylight.controller.cluster.common.actor.MessageTracker; import org.opendaylight.controller.cluster.common.actor.MessageTracker.Error; import org.opendaylight.controller.cluster.common.actor.MeteringBehavior; @@ -40,26 +70,35 @@ import org.opendaylight.controller.cluster.datastore.messages.CloseTransactionCh import org.opendaylight.controller.cluster.datastore.messages.CommitTransaction; import org.opendaylight.controller.cluster.datastore.messages.CreateTransaction; import org.opendaylight.controller.cluster.datastore.messages.CreateTransactionReply; -import org.opendaylight.controller.cluster.datastore.messages.DatastoreSnapshot; -import org.opendaylight.controller.cluster.datastore.messages.DatastoreSnapshot.ShardSnapshot; import org.opendaylight.controller.cluster.datastore.messages.ForwardedReadyTransaction; import org.opendaylight.controller.cluster.datastore.messages.GetShardDataTree; +import org.opendaylight.controller.cluster.datastore.messages.MakeLeaderLocal; +import org.opendaylight.controller.cluster.datastore.messages.OnDemandShardState; import org.opendaylight.controller.cluster.datastore.messages.PeerAddressResolved; +import org.opendaylight.controller.cluster.datastore.messages.PersistAbortTransactionPayload; import org.opendaylight.controller.cluster.datastore.messages.ReadyLocalTransaction; import org.opendaylight.controller.cluster.datastore.messages.RegisterChangeListener; import org.opendaylight.controller.cluster.datastore.messages.RegisterDataTreeChangeListener; import org.opendaylight.controller.cluster.datastore.messages.ShardLeaderStateChanged; import org.opendaylight.controller.cluster.datastore.messages.UpdateSchemaContext; -import org.opendaylight.controller.cluster.datastore.utils.Dispatchers; +import org.opendaylight.controller.cluster.datastore.persisted.AbortTransactionPayload; +import org.opendaylight.controller.cluster.datastore.persisted.DatastoreSnapshot; +import org.opendaylight.controller.cluster.datastore.persisted.DatastoreSnapshot.ShardSnapshot; +import org.opendaylight.controller.cluster.messaging.MessageAssembler; +import org.opendaylight.controller.cluster.messaging.MessageSlicer; +import org.opendaylight.controller.cluster.messaging.SliceOptions; import org.opendaylight.controller.cluster.notifications.LeaderStateChanged; import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListener; import org.opendaylight.controller.cluster.notifications.RoleChangeNotifier; +import org.opendaylight.controller.cluster.raft.LeadershipTransferFailedException; import org.opendaylight.controller.cluster.raft.RaftActor; import org.opendaylight.controller.cluster.raft.RaftActorRecoveryCohort; import org.opendaylight.controller.cluster.raft.RaftActorSnapshotCohort; import org.opendaylight.controller.cluster.raft.RaftState; import org.opendaylight.controller.cluster.raft.base.messages.FollowerInitialSyncUpStatus; +import org.opendaylight.controller.cluster.raft.client.messages.OnDemandRaftState; import org.opendaylight.controller.cluster.raft.messages.AppendEntriesReply; +import org.opendaylight.controller.cluster.raft.messages.RequestLeadership; import org.opendaylight.controller.cluster.raft.messages.ServerRemoved; import org.opendaylight.controller.cluster.raft.protobuff.client.messages.Payload; import org.opendaylight.yangtools.concepts.Identifier; @@ -67,12 +106,14 @@ import org.opendaylight.yangtools.yang.data.api.schema.tree.DataValidationFailed import org.opendaylight.yangtools.yang.data.api.schema.tree.TipProducingDataTree; import org.opendaylight.yangtools.yang.data.api.schema.tree.TreeType; import org.opendaylight.yangtools.yang.model.api.SchemaContext; +import org.opendaylight.yangtools.yang.model.api.SchemaContextProvider; import scala.concurrent.duration.Duration; import scala.concurrent.duration.FiniteDuration; /** * A Shard represents a portion of the logical data tree. - *

+ * + *

* Our Shard uses InMemoryDataTree as it's internal representation and delegates all requests it */ public class Shard extends RaftActor { @@ -93,9 +134,27 @@ public class Shard extends RaftActor { } }; + static final Object RESUME_NEXT_PENDING_TRANSACTION = new Object() { + @Override + public String toString() { + return "resumeNextPendingTransaction"; + } + }; + // FIXME: shard names should be encapsulated in their own class and this should be exposed as a constant. public static final String DEFAULT_NAME = "default"; + private static final Collection SUPPORTED_ABIVERSIONS; + + static { + final ABIVersion[] values = ABIVersion.values(); + final ABIVersion[] real = Arrays.copyOfRange(values, 1, values.length - 1); + SUPPORTED_ABIVERSIONS = ImmutableList.copyOf(real).reverse(); + } + + // FIXME: make this a dynamic property based on mailbox size and maximum number of clients + private static final int CLIENT_MAX_MESSAGES = 1000; + // The state of this Shard private final ShardDataTree store; @@ -128,7 +187,14 @@ public class Shard extends RaftActor { private final ShardTransactionMessageRetrySupport messageRetrySupport; - private final FrontendMetadata frontendMetadata = new FrontendMetadata(); + private final FrontendMetadata frontendMetadata; + private Map knownFrontends = ImmutableMap.of(); + private boolean paused; + + private final MessageSlicer responseMessageSlicer; + private final Dispatchers dispatchers; + + private final MessageAssembler requestMessageAssembler; protected Shard(final AbstractBuilder builder) { super(builder.getId().toString(), builder.getPeerAddresses(), @@ -137,21 +203,23 @@ public class Shard extends RaftActor { this.name = builder.getId().toString(); this.datastoreContext = builder.getDatastoreContext(); this.restoreFromSnapshot = builder.getRestoreFromSnapshot(); + this.frontendMetadata = new FrontendMetadata(name); setPersistence(datastoreContext.isPersistent()); LOG.info("Shard created : {}, persistent : {}", name, datastoreContext.isPersistent()); ShardDataTreeChangeListenerPublisherActorProxy treeChangeListenerPublisher = - new ShardDataTreeChangeListenerPublisherActorProxy(getContext(), name + "-DTCL-publisher"); + new ShardDataTreeChangeListenerPublisherActorProxy(getContext(), name + "-DTCL-publisher", name); ShardDataChangeListenerPublisherActorProxy dataChangeListenerPublisher = - new ShardDataChangeListenerPublisherActorProxy(getContext(), name + "-DCL-publisher"); + new ShardDataChangeListenerPublisherActorProxy(getContext(), name + "-DCL-publisher", name); if (builder.getDataTree() != null) { store = new ShardDataTree(this, builder.getSchemaContext(), builder.getDataTree(), - treeChangeListenerPublisher, dataChangeListenerPublisher, name); + treeChangeListenerPublisher, dataChangeListenerPublisher, name, frontendMetadata); } else { store = new ShardDataTree(this, builder.getSchemaContext(), builder.getTreeType(), - treeChangeListenerPublisher, dataChangeListenerPublisher, name); + builder.getDatastoreContext().getStoreRoot(), treeChangeListenerPublisher, + dataChangeListenerPublisher, name, frontendMetadata); } shardMBean = ShardMBeanFactory.getShardStatsMBean(name, datastoreContext.getDataStoreMXBeanType(), this); @@ -170,14 +238,25 @@ public class Shard extends RaftActor { appendEntriesReplyTracker = new MessageTracker(AppendEntriesReply.class, getRaftActorContext().getConfigParams().getIsolatedCheckIntervalInMillis()); + dispatchers = new Dispatchers(context().system().dispatchers()); transactionActorFactory = new ShardTransactionActorFactory(store, datastoreContext, - new Dispatchers(context().system().dispatchers()).getDispatcherPath( - Dispatchers.DispatcherType.Transaction), self(), getContext(), shardMBean); + dispatchers.getDispatcherPath(Dispatchers.DispatcherType.Transaction), + self(), getContext(), shardMBean, builder.getId().getShardName()); snapshotCohort = ShardSnapshotCohort.create(getContext(), builder.getId().getMemberName(), store, LOG, this.name); messageRetrySupport = new ShardTransactionMessageRetrySupport(this); + + responseMessageSlicer = MessageSlicer.builder().logContext(this.name) + .messageSliceSize(datastoreContext.getMaximumMessageSliceSize()) + .fileBackedStreamFactory(getRaftActorContext().getFileBackedOutputStreamFactory()) + .expireStateAfterInactivity(2, TimeUnit.MINUTES).build(); + + requestMessageAssembler = MessageAssembler.builder().logContext(this.name) + .fileBackedStreamFactory(getRaftActorContext().getFileBackedOutputStreamFactory()) + .assembledMessageCallback((message, sender) -> self().tell(message, sender)) + .expireStateAfterInactivity(datastoreContext.getRequestTimeout(), TimeUnit.NANOSECONDS).build(); } private void setTransactionCommitTimeout() { @@ -221,14 +300,22 @@ public class Shard extends RaftActor { @Override protected void handleNonRaftCommand(final Object message) { - try (final MessageTracker.Context context = appendEntriesReplyTracker.received(message)) { + try (MessageTracker.Context context = appendEntriesReplyTracker.received(message)) { final Optional maybeError = context.error(); if (maybeError.isPresent()) { LOG.trace("{} : AppendEntriesReply failed to arrive at the expected interval {}", persistenceId(), maybeError.get()); } - if (CreateTransaction.isSerializedType(message)) { + store.resetTransactionBatch(); + + if (message instanceof RequestEnvelope) { + handleRequestEnvelope((RequestEnvelope)message); + } else if (requestMessageAssembler.isHandledMessage(message)) { + handleRequestAssemblerMessage(message); + } else if (message instanceof ConnectClientRequest) { + handleConnectClient((ConnectClientRequest)message); + } else if (CreateTransaction.isSerializedType(message)) { handleCreateTransaction(message); } else if (message instanceof BatchedModifications) { handleBatchedModifications((BatchedModifications)message); @@ -254,8 +341,7 @@ public class Shard extends RaftActor { PeerAddressResolved resolved = (PeerAddressResolved) message; setPeerAddress(resolved.getPeerId(), resolved.getPeerAddress()); } else if (TX_COMMIT_TIMEOUT_CHECK_MESSAGE.equals(message)) { - store.checkForExpiredTransactions(transactionCommitTimeout); - commitCoordinator.checkForExpiredTransactions(transactionCommitTimeout, this); + commitTimeoutCheck(); } else if (message instanceof DatastoreContext) { onDatastoreContext((DatastoreContext)message); } else if (message instanceof RegisterRoleChangeListener) { @@ -274,12 +360,213 @@ public class Shard extends RaftActor { } else if (message instanceof DataTreeCohortActorRegistry.CohortRegistryCommand) { store.processCohortRegistryCommand(getSender(), (DataTreeCohortActorRegistry.CohortRegistryCommand) message); - } else { + } else if (message instanceof PersistAbortTransactionPayload) { + final TransactionIdentifier txId = ((PersistAbortTransactionPayload) message).getTransactionId(); + persistPayload(txId, AbortTransactionPayload.create(txId), true); + } else if (message instanceof MakeLeaderLocal) { + onMakeLeaderLocal(); + } else if (RESUME_NEXT_PENDING_TRANSACTION.equals(message)) { + store.resumeNextPendingTransaction(); + } else if (!responseMessageSlicer.handleMessage(message)) { super.handleNonRaftCommand(message); } } } + private void handleRequestAssemblerMessage(Object message) { + dispatchers.getDispatcher(DispatcherType.Serialization).execute(() -> { + JavaSerializer.currentSystem().value_$eq((ExtendedActorSystem) context().system()); + requestMessageAssembler.handleMessage(message, self()); + }); + } + + @SuppressWarnings("checkstyle:IllegalCatch") + private void handleRequestEnvelope(final RequestEnvelope envelope) { + final long now = ticker().read(); + try { + final RequestSuccess success = handleRequest(envelope, now); + if (success != null) { + final long executionTimeNanos = ticker().read() - now; + if (success instanceof SliceableMessage) { + dispatchers.getDispatcher(DispatcherType.Serialization).execute(() -> + responseMessageSlicer.slice(SliceOptions.builder().identifier(success.getTarget()) + .message(envelope.newSuccessEnvelope(success, executionTimeNanos)) + .sendTo(envelope.getMessage().getReplyTo()).replyTo(self()) + .onFailureCallback(t -> { + LOG.warn("Error slicing response {}", success, t); + }).build())); + } else { + envelope.sendSuccess(success, executionTimeNanos); + } + } + } catch (RequestException e) { + LOG.debug("{}: request {} failed", persistenceId(), envelope, e); + envelope.sendFailure(e, ticker().read() - now); + } catch (Exception e) { + LOG.debug("{}: request {} caused failure", persistenceId(), envelope, e); + envelope.sendFailure(new RuntimeRequestException("Request failed to process", e), + ticker().read() - now); + } + } + + private void commitTimeoutCheck() { + store.checkForExpiredTransactions(transactionCommitTimeout, this::updateAccess); + commitCoordinator.checkForExpiredTransactions(transactionCommitTimeout, this); + requestMessageAssembler.checkExpiredAssembledMessageState(); + } + + private Optional updateAccess(final SimpleShardDataTreeCohort cohort) { + final FrontendIdentifier frontend = cohort.getIdentifier().getHistoryId().getClientId().getFrontendId(); + final LeaderFrontendState state = knownFrontends.get(frontend); + if (state == null) { + // Not tell-based protocol, do nothing + return Optional.absent(); + } + + if (isIsolatedLeader()) { + // We are isolated and no new request can come through until we emerge from it. We are still updating + // liveness of frontend when we see it attempting to communicate. Use the last access timer. + return Optional.of(state.getLastSeenTicks()); + } + + // If this frontend has freshly connected, give it some time to catch up before killing its transactions. + return Optional.of(state.getLastConnectTicks()); + } + + private void onMakeLeaderLocal() { + LOG.debug("{}: onMakeLeaderLocal received", persistenceId()); + if (isLeader()) { + getSender().tell(new Status.Success(null), getSelf()); + return; + } + + final ActorSelection leader = getLeader(); + + if (leader == null) { + // Leader is not present. The cluster is most likely trying to + // elect a leader and we should let that run its normal course + + // TODO we can wait for the election to complete and retry the + // request. We can also let the caller retry by sending a flag + // in the response indicating the request is "reTryable". + getSender().tell(new Failure( + new LeadershipTransferFailedException("We cannot initiate leadership transfer to local node. " + + "Currently there is no leader for " + persistenceId())), + getSelf()); + return; + } + + leader.tell(new RequestLeadership(getId(), getSender()), getSelf()); + } + + // Acquire our frontend tracking handle and verify generation matches + @Nullable + private LeaderFrontendState findFrontend(final ClientIdentifier clientId) throws RequestException { + final LeaderFrontendState existing = knownFrontends.get(clientId.getFrontendId()); + if (existing != null) { + final int cmp = Long.compareUnsigned(existing.getIdentifier().getGeneration(), clientId.getGeneration()); + if (cmp == 0) { + existing.touch(); + return existing; + } + if (cmp > 0) { + LOG.debug("{}: rejecting request from outdated client {}", persistenceId(), clientId); + throw new RetiredGenerationException(existing.getIdentifier().getGeneration()); + } + + LOG.info("{}: retiring state {}, outdated by request from client {}", persistenceId(), existing, clientId); + existing.retire(); + knownFrontends.remove(clientId.getFrontendId()); + } else { + LOG.debug("{}: client {} is not yet known", persistenceId(), clientId); + } + + return null; + } + + private LeaderFrontendState getFrontend(final ClientIdentifier clientId) throws RequestException { + final LeaderFrontendState ret = findFrontend(clientId); + if (ret != null) { + return ret; + } + + // TODO: a dedicated exception would be better, but this is technically true, too + throw new OutOfSequenceEnvelopeException(0); + } + + private static @Nonnull ABIVersion selectVersion(final ConnectClientRequest message) { + final Range clientRange = Range.closed(message.getMinVersion(), message.getMaxVersion()); + for (ABIVersion v : SUPPORTED_ABIVERSIONS) { + if (clientRange.contains(v)) { + return v; + } + } + + throw new IllegalArgumentException(String.format( + "No common version between backend versions %s and client versions %s", SUPPORTED_ABIVERSIONS, + clientRange)); + } + + @SuppressWarnings("checkstyle:IllegalCatch") + private void handleConnectClient(final ConnectClientRequest message) { + try { + final ClientIdentifier clientId = message.getTarget(); + final LeaderFrontendState existing = findFrontend(clientId); + if (existing != null) { + existing.touch(); + } + + if (!isLeader() || !isLeaderActive()) { + LOG.info("{}: not currently leader, rejecting request {}. isLeader: {}, isLeaderActive: {}," + + "isLeadershipTransferInProgress: {}.", + persistenceId(), message, isLeader(), isLeaderActive(), isLeadershipTransferInProgress()); + throw new NotLeaderException(getSelf()); + } + + final ABIVersion selectedVersion = selectVersion(message); + final LeaderFrontendState frontend; + if (existing == null) { + frontend = new LeaderFrontendState(persistenceId(), clientId, store); + knownFrontends.put(clientId.getFrontendId(), frontend); + LOG.debug("{}: created state {} for client {}", persistenceId(), frontend, clientId); + } else { + frontend = existing; + } + + frontend.reconnect(); + message.getReplyTo().tell(new ConnectClientSuccess(message.getTarget(), message.getSequence(), getSelf(), + ImmutableList.of(), store.getDataTree(), CLIENT_MAX_MESSAGES).toVersion(selectedVersion), + ActorRef.noSender()); + } catch (RequestException | RuntimeException e) { + message.getReplyTo().tell(new Failure(e), ActorRef.noSender()); + } + } + + private @Nullable RequestSuccess handleRequest(final RequestEnvelope envelope, final long now) + throws RequestException { + // We are not the leader, hence we want to fail-fast. + if (!isLeader() || paused || !isLeaderActive()) { + LOG.debug("{}: not currently active leader, rejecting request {}. isLeader: {}, isLeaderActive: {}," + + "isLeadershipTransferInProgress: {}, paused: {}", + persistenceId(), envelope, isLeader(), isLeaderActive(), isLeadershipTransferInProgress(), paused); + throw new NotLeaderException(getSelf()); + } + + final Request request = envelope.getMessage(); + if (request instanceof TransactionRequest) { + final TransactionRequest txReq = (TransactionRequest)request; + final ClientIdentifier clientId = txReq.getTarget().getHistoryId().getClientId(); + return getFrontend(clientId).handleTransactionRequest(txReq, envelope, now); + } else if (request instanceof LocalHistoryRequest) { + final LocalHistoryRequest lhReq = (LocalHistoryRequest)request; + final ClientIdentifier clientId = lhReq.getTarget().getClientId(); + return getFrontend(clientId).handleLocalHistoryRequest(lhReq, envelope, now); + } else { + LOG.warn("{}: rejecting unsupported request {}", persistenceId(), request); + throw new UnsupportedRequestException(request); + } + } + private boolean hasLeader() { return getLeaderId() != null; } @@ -314,16 +601,15 @@ public class Shard extends RaftActor { updateConfigParams(datastoreContext.getShardRaftConfig()); } - boolean canSkipPayload() { - // If we do not have any followers and we are not using persistence we can apply modification to the state - // immediately - return !hasFollowers() && !persistence().isRecoveryApplicable(); - } - // applyState() will be invoked once consensus is reached on the payload - void persistPayload(final TransactionIdentifier transactionId, final Payload payload) { - // We are faking the sender - persistData(self(), transactionId, payload); + void persistPayload(final Identifier id, final Payload payload, final boolean batchHint) { + boolean canSkipPayload = !hasFollowers() && !persistence().isRecoveryApplicable(); + if (canSkipPayload) { + applyState(self(), id, payload); + } else { + // We are faking the sender + persistData(self(), id, payload, batchHint); + } } private void handleCommitTransaction(final CommitTransaction commit) { @@ -365,7 +651,7 @@ public class Shard extends RaftActor { } catch (Exception e) { LOG.error("{}: Error handling BatchedModifications for Tx {}", persistenceId(), batched.getTransactionId(), e); - sender.tell(new akka.actor.Status.Failure(e), getSelf()); + sender.tell(new Failure(e), getSelf()); } } @@ -411,7 +697,7 @@ public class Shard extends RaftActor { private boolean failIfIsolatedLeader(final ActorRef sender) { if (isIsolatedLeader()) { - sender.tell(new akka.actor.Status.Failure(new NoShardLeaderException(String.format( + sender.tell(new Failure(new NoShardLeaderException(String.format( "Shard %s was the leader but has lost contact with all of its followers. Either all" + " other follower nodes are down or this node is isolated by a network partition.", persistenceId()))), getSelf()); @@ -436,7 +722,7 @@ public class Shard extends RaftActor { } catch (Exception e) { LOG.error("{}: Error handling ReadyLocalTransaction for Tx {}", persistenceId(), message.getTransactionId(), e); - getSender().tell(new akka.actor.Status.Failure(e), getSelf()); + getSender().tell(new Failure(e), getSelf()); } } else { ActorSelection leader = getLeader(); @@ -477,7 +763,7 @@ public class Shard extends RaftActor { doAbortTransaction(abort.getTransactionId(), getSender()); } - void doAbortTransaction(final TransactionIdentifier transactionID, final ActorRef sender) { + void doAbortTransaction(final Identifier transactionID, final ActorRef sender) { commitCoordinator.handleAbort(transactionID, sender, this); } @@ -487,13 +773,15 @@ public class Shard extends RaftActor { } else if (getLeader() != null) { getLeader().forward(message, getContext()); } else { - getSender().tell(new akka.actor.Status.Failure(new NoShardLeaderException( + getSender().tell(new Failure(new NoShardLeaderException( "Could not create a shard transaction", persistenceId())), getSelf()); } } private void closeTransactionChain(final CloseTransactionChain closeTransactionChain) { - store.closeTransactionChain(closeTransactionChain.getIdentifier()); + final LocalHistoryIdentifier id = closeTransactionChain.getIdentifier(); + store.closeTransactionChain(id, null); + store.purgeTransactionChain(id, null); } @SuppressWarnings("checkstyle:IllegalCatch") @@ -510,7 +798,7 @@ public class Shard extends RaftActor { getSender().tell(new CreateTransactionReply(Serialization.serializedActorPath(transactionActor), createTransaction.getTransactionId(), createTransaction.getVersion()).toSerializable(), getSelf()); } catch (Exception e) { - getSender().tell(new akka.actor.Status.Failure(e), getSelf()); + getSender().tell(new Failure(e), getSelf()); } } @@ -593,7 +881,8 @@ public class Shard extends RaftActor { persistenceId(), getId()); } - store.closeAllTransactionChains(); + paused = false; + store.purgeLeaderState(); } if (hasLeader && !isIsolatedLeader()) { @@ -604,9 +893,21 @@ public class Shard extends RaftActor { @Override protected void onLeaderChanged(final String oldLeader, final String newLeader) { shardMBean.incrementLeadershipChangeCount(); + paused = false; + + if (!isLeader()) { + if (!knownFrontends.isEmpty()) { + LOG.debug("{}: removing frontend state for {}", persistenceId(), knownFrontends.keySet()); + knownFrontends = ImmutableMap.of(); + } + + requestMessageAssembler.close(); + + if (!hasLeader()) { + // No leader anywhere, nothing else to do + return; + } - boolean hasLeader = hasLeader(); - if (hasLeader && !isLeader()) { // Another leader was elected. If we were the previous leader and had pending transactions, convert // them to transaction messages and send to the new leader. ActorSelection leader = getLeader(); @@ -625,9 +926,13 @@ public class Shard extends RaftActor { commitCoordinator.abortPendingTransactions("The transacton was aborted due to inflight leadership " + "change and the leader address isn't available.", this); } + } else { + // We have become the leader, we need to reconstruct frontend state + knownFrontends = Verify.verifyNotNull(frontendMetadata.toLeaderState(this)); + LOG.debug("{}: became leader with frontend state for {}", persistenceId(), knownFrontends.keySet()); } - if (hasLeader && !isIsolatedLeader()) { + if (!isIsolatedLeader()) { messageRetrySupport.retryMessages(); } } @@ -645,9 +950,33 @@ public class Shard extends RaftActor { @Override protected void pauseLeader(final Runnable operation) { LOG.debug("{}: In pauseLeader, operation: {}", persistenceId(), operation); + paused = true; + + // Tell-based protocol can replay transaction state, so it is safe to blow it up when we are paused. + knownFrontends.values().forEach(LeaderFrontendState::retire); + knownFrontends = ImmutableMap.of(); + store.setRunOnPendingTransactionsComplete(operation); } + @Override + protected void unpauseLeader() { + LOG.debug("{}: In unpauseLeader", persistenceId()); + paused = false; + + store.setRunOnPendingTransactionsComplete(null); + + // Restore tell-based protocol state as if we were becoming the leader + knownFrontends = Verify.verifyNotNull(frontendMetadata.toLeaderState(this)); + } + + @Override + protected OnDemandRaftState.AbstractBuilder newOnDemandRaftStateBuilder() { + return OnDemandShardState.newBuilder().treeChangeListenerActors(treeChangeSupport.getListenerActors()) + .dataChangeListenerActors(changeSupport.getListenerActors()) + .commitCohortActors(store.getCohortActors()); + } + @Override public String persistenceId() { return this.name; @@ -681,7 +1010,7 @@ public class Shard extends RaftActor { private ShardIdentifier id; private Map peerAddresses = Collections.emptyMap(); private DatastoreContext datastoreContext; - private SchemaContext schemaContext; + private SchemaContextProvider schemaContextProvider; private DatastoreSnapshot.ShardSnapshot restoreFromSnapshot; private TipProducingDataTree dataTree; private volatile boolean sealed; @@ -717,9 +1046,9 @@ public class Shard extends RaftActor { return self(); } - public T schemaContext(final SchemaContext newSchemaContext) { + public T schemaContextProvider(final SchemaContextProvider schemaContextProvider) { checkSealed(); - this.schemaContext = newSchemaContext; + this.schemaContextProvider = Preconditions.checkNotNull(schemaContextProvider); return self(); } @@ -748,7 +1077,7 @@ public class Shard extends RaftActor { } public SchemaContext getSchemaContext() { - return schemaContext; + return Verify.verifyNotNull(schemaContextProvider.getSchemaContext()); } public DatastoreSnapshot.ShardSnapshot getRestoreFromSnapshot() { @@ -775,7 +1104,7 @@ public class Shard extends RaftActor { Preconditions.checkNotNull(id, "id should not be null"); Preconditions.checkNotNull(peerAddresses, "peerAddresses should not be null"); Preconditions.checkNotNull(datastoreContext, "dataStoreContext should not be null"); - Preconditions.checkNotNull(schemaContext, "schemaContext should not be null"); + Preconditions.checkNotNull(schemaContextProvider, "schemaContextProvider should not be null"); } public Props props() { @@ -794,4 +1123,8 @@ public class Shard extends RaftActor { Ticker ticker() { return Ticker.systemTicker(); } + + void scheduleNextPendingTransaction() { + self().tell(RESUME_NEXT_PENDING_TRANSACTION, ActorRef.noSender()); + } }