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=7ca79fc2349add0e9c19236a945aaed53676b45d;hp=4a1be9862b7fb6114511c068dd255f17e4b59a1e;hb=43fb391bf873b252383a8d736b2651b04da8d40d;hpb=5fd8e6506248cc34da72281a1662612f6c2b2f9a 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 4a1be9862b..7ca79fc234 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 @@ -37,6 +37,7 @@ import org.opendaylight.controller.cluster.access.commands.NotLeaderException; 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; @@ -61,16 +62,17 @@ 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.OnDemandShardState; import org.opendaylight.controller.cluster.datastore.messages.PeerAddressResolved; 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.persisted.DatastoreSnapshot; +import org.opendaylight.controller.cluster.datastore.persisted.DatastoreSnapshot.ShardSnapshot; import org.opendaylight.controller.cluster.datastore.utils.Dispatchers; import org.opendaylight.controller.cluster.notifications.LeaderStateChanged; import org.opendaylight.controller.cluster.notifications.RegisterRoleChangeListener; @@ -80,6 +82,7 @@ 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.ServerRemoved; import org.opendaylight.controller.cluster.raft.protobuff.client.messages.Payload; @@ -93,7 +96,8 @@ 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 { @@ -184,7 +188,8 @@ public class Shard extends RaftActor { treeChangeListenerPublisher, dataChangeListenerPublisher, name); } else { store = new ShardDataTree(this, builder.getSchemaContext(), builder.getTreeType(), - treeChangeListenerPublisher, dataChangeListenerPublisher, name); + builder.getDatastoreContext().getStoreRoot(), treeChangeListenerPublisher, + dataChangeListenerPublisher, name); } shardMBean = ShardMBeanFactory.getShardStatsMBean(name, datastoreContext.getDataStoreMXBeanType(), this); @@ -204,8 +209,8 @@ public class Shard extends RaftActor { getRaftActorContext().getConfigParams().getIsolatedCheckIntervalInMillis()); transactionActorFactory = new ShardTransactionActorFactory(store, datastoreContext, - new Dispatchers(context().system().dispatchers()).getDispatcherPath( - Dispatchers.DispatcherType.Transaction), self(), getContext(), shardMBean); + new Dispatchers(context().system().dispatchers()).getDispatcherPath(Dispatchers.DispatcherType.Transaction), + self(), getContext(), shardMBean, builder.getId().getShardName()); snapshotCohort = ShardSnapshotCohort.create(getContext(), builder.getId().getMemberName(), store, LOG, this.name); @@ -255,7 +260,7 @@ public class Shard extends RaftActor { @SuppressWarnings("checkstyle:IllegalCatch") @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(), @@ -263,18 +268,21 @@ public class Shard extends RaftActor { } if (message instanceof RequestEnvelope) { + final long now = ticker().read(); final RequestEnvelope envelope = (RequestEnvelope)message; + try { - final RequestSuccess success = handleRequest(envelope); + final RequestSuccess success = handleRequest(envelope, now); if (success != null) { - envelope.sendSuccess(success); + envelope.sendSuccess(success, ticker().read() - now); } } catch (RequestException e) { LOG.debug("{}: request {} failed", persistenceId(), envelope, e); - envelope.sendFailure(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)); + envelope.sendFailure(new RuntimeRequestException("Request failed to process", e), + ticker().read() - now); } } else if (message instanceof ConnectClientRequest) { handleConnectClient((ConnectClientRequest)message); @@ -388,7 +396,8 @@ public class Shard extends RaftActor { } } - private @Nullable RequestSuccess handleRequest(final RequestEnvelope envelope) throws RequestException { + 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() || !isLeaderActive()) { LOG.debug("{}: not currently leader, rejecting request {}", persistenceId(), envelope); @@ -399,11 +408,11 @@ public class Shard extends RaftActor { if (request instanceof TransactionRequest) { final TransactionRequest txReq = (TransactionRequest)request; final ClientIdentifier clientId = txReq.getTarget().getHistoryId().getClientId(); - return getFrontend(clientId).handleTransactionRequest(txReq, envelope); + 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); + return getFrontend(clientId).handleLocalHistoryRequest(lhReq, envelope, now); } else { LOG.debug("{}: rejecting unsupported request {}", persistenceId(), request); throw new UnsupportedRequestException(request); @@ -444,16 +453,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) { @@ -607,7 +615,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); } @@ -623,7 +631,9 @@ public class Shard extends RaftActor { } 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") @@ -778,6 +788,13 @@ public class Shard extends RaftActor { store.setRunOnPendingTransactionsComplete(operation); } + @Override + protected OnDemandRaftState.AbstractBuilder newOnDemandRaftStateBuilder() { + return OnDemandShardState.newBuilder().treeChangeListenerActors(treeChangeSupport.getListenerActors()) + .dataChangeListenerActors(changeSupport.getListenerActors()) + .commitCohortActors(store.getCohortActors()); + } + @Override public String persistenceId() { return this.name;