Refactor DataStore readiness tracking
[controller.git] / opendaylight / md-sal / sal-distributed-datastore / src / main / java / org / opendaylight / controller / cluster / datastore / AbstractDataStore.java
index 55108e0ef9cd720895b04d560145022b3cdae703..95cd0adc8aa48b71de1b9befe6248637cb9e3e2b 100644 (file)
@@ -7,17 +7,23 @@
  */
 package org.opendaylight.controller.cluster.datastore;
 
+import static com.google.common.base.Preconditions.checkArgument;
 import static java.util.Objects.requireNonNull;
 
 import akka.actor.ActorRef;
 import akka.actor.ActorSystem;
 import akka.actor.PoisonPill;
 import akka.actor.Props;
+import com.google.common.annotations.Beta;
 import com.google.common.annotations.VisibleForTesting;
 import com.google.common.base.Throwables;
+import com.google.common.util.concurrent.ListenableFuture;
+import com.google.common.util.concurrent.SettableFuture;
 import com.google.common.util.concurrent.Uninterruptibles;
-import java.util.concurrent.CountDownLatch;
+import java.util.Set;
+import java.util.concurrent.ExecutionException;
 import java.util.concurrent.TimeUnit;
+import java.util.concurrent.TimeoutException;
 import org.opendaylight.controller.cluster.access.concepts.ClientIdentifier;
 import org.opendaylight.controller.cluster.common.actor.Dispatchers;
 import org.opendaylight.controller.cluster.databroker.actors.dds.DataStoreClient;
@@ -27,6 +33,7 @@ import org.opendaylight.controller.cluster.datastore.identifiers.ShardManagerIde
 import org.opendaylight.controller.cluster.datastore.jmx.mbeans.DatastoreConfigurationMXBeanImpl;
 import org.opendaylight.controller.cluster.datastore.jmx.mbeans.DatastoreInfoMXBeanImpl;
 import org.opendaylight.controller.cluster.datastore.persisted.DatastoreSnapshot;
+import org.opendaylight.controller.cluster.datastore.shardmanager.AbstractShardManagerCreator;
 import org.opendaylight.controller.cluster.datastore.shardmanager.ShardManagerCreator;
 import org.opendaylight.controller.cluster.datastore.utils.ActorUtils;
 import org.opendaylight.controller.cluster.datastore.utils.ClusterUtils;
@@ -40,36 +47,30 @@ import org.opendaylight.mdsal.dom.api.DOMDataTreeIdentifier;
 import org.opendaylight.mdsal.dom.spi.store.DOMStoreTreeChangePublisher;
 import org.opendaylight.yangtools.concepts.ListenerRegistration;
 import org.opendaylight.yangtools.yang.data.api.YangInstanceIdentifier;
-import org.opendaylight.yangtools.yang.model.api.SchemaContext;
-import org.opendaylight.yangtools.yang.model.api.SchemaContextListener;
+import org.opendaylight.yangtools.yang.model.api.EffectiveModelContext;
+import org.opendaylight.yangtools.yang.model.api.EffectiveModelContextListener;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
+import scala.concurrent.duration.Duration;
 
 /**
  * Base implementation of a distributed DOMStore.
  */
-public abstract class AbstractDataStore implements DistributedDataStoreInterface, SchemaContextListener,
+public abstract class AbstractDataStore implements DistributedDataStoreInterface, EffectiveModelContextListener,
         DatastoreContextPropertiesUpdater.Listener, DOMStoreTreeChangePublisher,
         DOMDataTreeCommitCohortRegistry, AutoCloseable {
 
     private static final Logger LOG = LoggerFactory.getLogger(AbstractDataStore.class);
 
-    private static final long READY_WAIT_FACTOR = 3;
-
+    private final SettableFuture<Void> readinessFuture = SettableFuture.create();
+    private final ClientIdentifier identifier;
+    private final DataStoreClient client;
     private final ActorUtils actorUtils;
-    private final long waitTillReadyTimeInMillis;
 
     private AutoCloseable closeable;
-
     private DatastoreConfigurationMXBeanImpl datastoreConfigMXBean;
-
     private DatastoreInfoMXBeanImpl datastoreInfoMXBean;
 
-    private final CountDownLatch waitTillReadyCountDownLatch = new CountDownLatch(1);
-
-    private final ClientIdentifier identifier;
-    private final DataStoreClient client;
-
     @SuppressWarnings("checkstyle:IllegalCatch")
     protected AbstractDataStore(final ActorSystem actorSystem, final ClusterWrapper cluster,
             final Configuration configuration, final DatastoreContextFactory datastoreContextFactory,
@@ -89,9 +90,9 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
 
         PrimaryShardInfoFutureCache primaryShardInfoCache = new PrimaryShardInfoFutureCache();
 
-        ShardManagerCreator creator = new ShardManagerCreator().cluster(cluster).configuration(configuration)
+        AbstractShardManagerCreator<?> creator = getShardManagerCreator().cluster(cluster).configuration(configuration)
                 .datastoreContextFactory(datastoreContextFactory)
-                .waitTillReadyCountDownLatch(waitTillReadyCountDownLatch)
+                .readinessFuture(readinessFuture)
                 .primaryShardInfoCache(primaryShardInfoCache)
                 .restoreFromSnapshot(restoreFromSnapshot)
                 .distributedDataStore(this);
@@ -115,9 +116,6 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         identifier = client.getIdentifier();
         LOG.debug("Distributed data store client {} started", identifier);
 
-        this.waitTillReadyTimeInMillis = actorUtils.getDatastoreContext().getShardLeaderElectionTimeout()
-                .duration().toMillis() * READY_WAIT_FACTOR;
-
         datastoreConfigMXBean = new DatastoreConfigurationMXBeanImpl(
                 datastoreContextFactory.getBaseDatastoreContext().getDataStoreMXBeanType());
         datastoreConfigMXBean.setContext(datastoreContextFactory.getBaseDatastoreContext());
@@ -133,8 +131,6 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         this.actorUtils = requireNonNull(actorUtils, "actorContext should not be null");
         this.client = null;
         this.identifier = requireNonNull(identifier);
-        this.waitTillReadyTimeInMillis = actorUtils.getDatastoreContext().getShardLeaderElectionTimeout()
-                .duration().toMillis() * READY_WAIT_FACTOR;
     }
 
     @VisibleForTesting
@@ -143,8 +139,10 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         this.actorUtils = requireNonNull(actorUtils, "actorContext should not be null");
         this.client = clientActor;
         this.identifier = requireNonNull(identifier);
-        this.waitTillReadyTimeInMillis = actorUtils.getDatastoreContext().getShardLeaderElectionTimeout()
-                .duration().toMillis() * READY_WAIT_FACTOR;
+    }
+
+    protected AbstractShardManagerCreator<?> getShardManagerCreator() {
+        return new ShardManagerCreator();
     }
 
     protected final DataStoreClient getClient() {
@@ -165,6 +163,21 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         requireNonNull(treeId, "treeId should not be null");
         requireNonNull(listener, "listener should not be null");
 
+        /*
+         * We need to potentially deal with multi-shard composition for registration targeting the root of the data
+         * store. If that is the case, we delegate to a more complicated setup invol
+         */
+        if (treeId.isEmpty()) {
+            // User is targeting root of the datastore. If there is more than one shard, we have to register with them
+            // all and perform data composition.
+            final Set<String> shardNames = actorUtils.getConfiguration().getAllShardNames();
+            if (shardNames.size() > 1) {
+                checkArgument(listener instanceof ClusteredDOMDataTreeChangeListener,
+                    "Cannot listen on root without non-clustered listener %s", listener);
+                return new RootDataTreeChangeListenerProxy<>(actorUtils, listener, shardNames);
+            }
+        }
+
         final String shardName = actorUtils.getShardStrategyFactory().getStrategy(treeId).findShard(treeId);
         LOG.debug("Registering tree listener: {} for tree: {} shard: {}", listener, treeId, shardName);
 
@@ -175,7 +188,6 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         return listenerRegistrationProxy;
     }
 
-
     @Override
     public <C extends DOMDataTreeCommitCohort> DOMDataTreeCommitCohortRegistration<C> registerCommitCohort(
             final DOMDataTreeIdentifier subtree, final C cohort) {
@@ -193,8 +205,8 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
     }
 
     @Override
-    public void onGlobalContextUpdated(final SchemaContext schemaContext) {
-        actorUtils.setSchemaContext(schemaContext);
+    public void onModelContextUpdated(final EffectiveModelContext newModelContext) {
+        actorUtils.setSchemaContext(newModelContext);
     }
 
     @Override
@@ -237,24 +249,64 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         return actorUtils;
     }
 
+    // TODO: consider removing this in favor of awaitReadiness()
+    @Deprecated
     public void waitTillReady() {
         LOG.info("Beginning to wait for data store to become ready : {}", identifier);
 
+        final Duration toWait = initialSettleTime();
         try {
-            if (waitTillReadyCountDownLatch.await(waitTillReadyTimeInMillis, TimeUnit.MILLISECONDS)) {
-                LOG.debug("Data store {} is now ready", identifier);
-            } else {
-                LOG.error("Shard leaders failed to settle in {} seconds, giving up",
-                        TimeUnit.MILLISECONDS.toSeconds(waitTillReadyTimeInMillis));
+            if (!awaitReadiness(toWait)) {
+                LOG.error("Shard leaders failed to settle in {}, giving up", toWait);
+                return;
             }
         } catch (InterruptedException e) {
             LOG.error("Interrupted while waiting for shards to settle", e);
+            return;
+        }
+
+        LOG.debug("Data store {} is now ready", identifier);
+    }
+
+    @Beta
+    @Deprecated
+    public boolean awaitReadiness() throws InterruptedException {
+        return awaitReadiness(initialSettleTime());
+    }
+
+    @Beta
+    @Deprecated
+    public boolean awaitReadiness(final Duration toWait) throws InterruptedException {
+        try {
+            if (toWait.isFinite()) {
+                try {
+                    readinessFuture.get(toWait.toNanos(), TimeUnit.NANOSECONDS);
+                } catch (TimeoutException e) {
+                    LOG.debug("Timed out waiting for shards to settle", e);
+                    return false;
+                }
+            } else {
+                readinessFuture.get();
+            }
+        } catch (ExecutionException e) {
+            LOG.warn("Unexpected readiness failure, assuming convergence", e);
+        }
+
+        return true;
+    }
+
+    @Beta
+    @Deprecated
+    public void awaitReadiness(final long timeout, final TimeUnit unit) throws InterruptedException, TimeoutException {
+        if (!awaitReadiness(Duration.create(timeout, unit))) {
+            throw new TimeoutException("Shard leaders failed to settle");
         }
     }
 
     @SuppressWarnings("checkstyle:IllegalCatch")
-    private static ActorRef createShardManager(final ActorSystem actorSystem, final ShardManagerCreator creator,
-            final String shardDispatcher, final String shardManagerId) {
+    private static ActorRef createShardManager(final ActorSystem actorSystem,
+            final AbstractShardManagerCreator<?> creator, final String shardDispatcher,
+            final String shardManagerId) {
         Exception lastException = null;
 
         for (int i = 0; i < 100; i++) {
@@ -271,16 +323,25 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         throw new IllegalStateException("Failed to create Shard Manager", lastException);
     }
 
+    /**
+     * Future which completes when all shards settle for the first time.
+     *
+     * @return A Listenable future.
+     */
+    public final ListenableFuture<?> initialSettleFuture() {
+        return readinessFuture;
+    }
+
     @VisibleForTesting
-    public CountDownLatch getWaitTillReadyCountDownLatch() {
-        return waitTillReadyCountDownLatch;
+    SettableFuture<Void> readinessFuture() {
+        return readinessFuture;
     }
 
+    @Override
     @SuppressWarnings("unchecked")
     public <L extends DOMDataTreeChangeListener> ListenerRegistration<L> registerProxyListener(
-            final YangInstanceIdentifier shardLookup,
-            final YangInstanceIdentifier insideShard,
-            final org.opendaylight.mdsal.dom.api.DOMDataTreeChangeListener delegate) {
+            final YangInstanceIdentifier shardLookup, final YangInstanceIdentifier insideShard,
+            final DOMDataTreeChangeListener delegate) {
 
         requireNonNull(shardLookup, "shardLookup should not be null");
         requireNonNull(insideShard, "insideShard should not be null");
@@ -300,10 +361,10 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         return (ListenerRegistration<L>) listenerRegistrationProxy;
     }
 
+    @Override
     @SuppressWarnings("unchecked")
     public <L extends DOMDataTreeChangeListener> ListenerRegistration<L> registerShardConfigListener(
-            final YangInstanceIdentifier internalPath,
-            final DOMDataTreeChangeListener delegate) {
+            final YangInstanceIdentifier internalPath, final DOMDataTreeChangeListener delegate) {
         requireNonNull(delegate, "delegate should not be null");
 
         LOG.debug("Registering a listener for the configuration shard: {}", internalPath);
@@ -315,4 +376,9 @@ public abstract class AbstractDataStore implements DistributedDataStoreInterface
         return (ListenerRegistration<L>) proxy;
     }
 
+    private Duration initialSettleTime() {
+        final DatastoreContext context = actorUtils.getDatastoreContext();
+        final int multiplier = context.getInitialSettleTimeoutMultiplier();
+        return multiplier == 0 ? Duration.Inf() : context.getShardLeaderElectionTimeout().duration().$times(multiplier);
+    }
 }