Fix unsynchronized LSP counting
[bgpcep.git] / pcep / topology / topology-provider / src / main / java / org / opendaylight / bgpcep / pcep / topology / provider / AbstractTopologySessionListener.java
index 22aa35ccc307c9aca705d82e55ed9006d5cfeecc..ccb4ccb8f2f8173111ab058ac3b4976be30f52f2 100755 (executable)
@@ -9,8 +9,9 @@ package org.opendaylight.bgpcep.pcep.topology.provider;
 
 import static java.util.Objects.requireNonNull;
 
-import com.google.common.base.Optional;
 import com.google.common.base.Preconditions;
+import com.google.common.collect.ImmutableList;
+import com.google.common.util.concurrent.FluentFuture;
 import com.google.common.util.concurrent.FutureCallback;
 import com.google.common.util.concurrent.Futures;
 import com.google.common.util.concurrent.ListenableFuture;
@@ -24,41 +25,43 @@ import java.util.List;
 import java.util.Map;
 import java.util.Map.Entry;
 import java.util.Objects;
+import java.util.Optional;
 import java.util.Timer;
 import java.util.TimerTask;
 import java.util.concurrent.TimeUnit;
 import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.stream.Stream;
 import javax.annotation.concurrent.GuardedBy;
 import org.opendaylight.bgpcep.pcep.topology.provider.session.stats.SessionStateImpl;
 import org.opendaylight.bgpcep.pcep.topology.provider.session.stats.TopologySessionStats;
-import org.opendaylight.controller.md.sal.binding.api.ReadWriteTransaction;
-import org.opendaylight.controller.md.sal.binding.api.WriteTransaction;
-import org.opendaylight.controller.md.sal.common.api.data.LogicalDatastoreType;
+import org.opendaylight.mdsal.binding.api.WriteTransaction;
+import org.opendaylight.mdsal.common.api.CommitInfo;
+import org.opendaylight.mdsal.common.api.LogicalDatastoreType;
 import org.opendaylight.protocol.pcep.PCEPCloseTermination;
 import org.opendaylight.protocol.pcep.PCEPSession;
 import org.opendaylight.protocol.pcep.PCEPTerminationReason;
 import org.opendaylight.protocol.pcep.TerminationReason;
 import org.opendaylight.yang.gen.v1.urn.ietf.params.xml.ns.yang.ietf.inet.types.rev130715.IpAddressBuilder;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.ietf.stateful.rev171025.LspObject;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.ietf.stateful.rev171025.Path1;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.ietf.stateful.rev171025.lsp.object.Lsp;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev131005.Message;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev131005.MessageHeader;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev131005.Object;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev131005.ProtocolVersion;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.LspId;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.Node1;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.Node1Builder;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.OperationResult;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.PccSyncState;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.TearDownSessionInput;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.lsp.metadata.Metadata;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.pcep.client.attributes.PathComputationClient;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.pcep.client.attributes.PathComputationClientBuilder;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.pcep.client.attributes.path.computation.client.ReportedLsp;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.pcep.client.attributes.path.computation.client.ReportedLspBuilder;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.pcep.client.attributes.path.computation.client.ReportedLspKey;
-import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev171025.pcep.client.attributes.path.computation.client.reported.lsp.Path;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.ietf.stateful.rev181109.LspObject;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.ietf.stateful.rev181109.Path1;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.ietf.stateful.rev181109.lsp.object.Lsp;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev181109.Message;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev181109.MessageHeader;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev181109.Object;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.pcep.types.rev181109.ProtocolVersion;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.LspId;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.Node1;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.Node1Builder;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.OperationResult;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.PccSyncState;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.TearDownSessionInput;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.lsp.metadata.Metadata;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.pcep.client.attributes.PathComputationClient;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.pcep.client.attributes.PathComputationClientBuilder;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.pcep.client.attributes.path.computation.client.ReportedLsp;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.pcep.client.attributes.path.computation.client.ReportedLspBuilder;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.pcep.client.attributes.path.computation.client.ReportedLspKey;
+import org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.topology.pcep.rev181109.pcep.client.attributes.path.computation.client.reported.lsp.Path;
 import org.opendaylight.yang.gen.v1.urn.tbd.params.xml.ns.yang.network.topology.rev131021.network.topology.topology.Node;
 import org.opendaylight.yangtools.yang.binding.DataContainer;
 import org.opendaylight.yangtools.yang.binding.DataObject;
@@ -102,10 +105,14 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
     private final Map<String, ReportedLsp> lspData = new HashMap<>();
     private final ServerSessionManager serverSessionManager;
     private InstanceIdentifier<PathComputationClient> pccIdentifier;
+    @GuardedBy("this")
     private TopologyNodeState nodeState;
-    private AtomicBoolean synced = new AtomicBoolean(false);
+    private final AtomicBoolean synced = new AtomicBoolean(false);
+    @GuardedBy("this")
     private PCEPSession session;
+    @GuardedBy("this")
     private SyncOptimization syncOptimization;
+    @GuardedBy("this")
     private boolean triggeredResyncInProcess;
 
     AbstractTopologySessionListener(final ServerSessionManager serverSessionManager) {
@@ -114,35 +121,35 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
     }
 
     @Override
-    public final synchronized void onSessionUp(final PCEPSession session) {
+    public final synchronized void onSessionUp(final PCEPSession psession) {
         /*
          * The session went up. Look up the router in Inventory model,
          * create it if it is not there (marking that fact for later
          * deletion), and mark it as synchronizing. Also create it in
          * the topology model, with empty LSP list.
          */
-        final InetAddress peerAddress = session.getRemoteAddress();
+        final InetAddress peerAddress = psession.getRemoteAddress();
 
-        this.syncOptimization = new SyncOptimization(session);
+        this.syncOptimization = new SyncOptimization(psession);
 
         final TopologyNodeState state = this.serverSessionManager.takeNodeState(peerAddress,
                 this, isLspDbRetreived());
 
         // takeNodeState(..) may fail when the server session manager is being restarted due to configuration change
         if (state == null) {
-            LOG.error("Unable to fetch topology node state for PCEP session. Closing session {}", session);
-            session.close(TerminationReason.UNKNOWN);
-            this.onSessionTerminated(session, new PCEPCloseTermination(TerminationReason.UNKNOWN));
+            LOG.error("Unable to fetch topology node state for PCEP session. Closing session {}", psession);
+            psession.close(TerminationReason.UNKNOWN);
+            this.onSessionTerminated(psession, new PCEPCloseTermination(TerminationReason.UNKNOWN));
             return;
         }
 
         if (this.session != null || this.nodeState != null) {
-            LOG.error("PCEP session is already up with {}. Closing session {}", session.getRemoteAddress(), session);
-            session.close(TerminationReason.UNKNOWN);
-            this.onSessionTerminated(session, new PCEPCloseTermination(TerminationReason.UNKNOWN));
+            LOG.error("PCEP session is already up with {}. Closing session {}", psession.getRemoteAddress(), psession);
+            psession.close(TerminationReason.UNKNOWN);
+            this.onSessionTerminated(psession, new PCEPCloseTermination(TerminationReason.UNKNOWN));
             return;
         }
-        this.session = session;
+        this.session = psession;
         this.nodeState = state;
         this.serverSessionManager.bind(this.nodeState.getNodeId(), this.listenerState);
 
@@ -151,7 +158,7 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
         // Our augmentation in the topology node
         final PathComputationClientBuilder pccBuilder = new PathComputationClientBuilder();
 
-        onSessionUp(session, pccBuilder);
+        onSessionUp(psession, pccBuilder);
         this.synced.set(isSynchronized());
 
         pccBuilder.setIpAddress(IpAddressBuilder.getDefaultInstance(peerAddress.getHostAddress()));
@@ -161,93 +168,67 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
         final boolean isNodePresent = isLspDbRetreived() && initialNodeState != null;
         if (isNodePresent) {
             loadLspData(initialNodeState, this.lspData, this.lsps, isIncrementalSynchro());
-            pccBuilder.setReportedLsp(initialNodeState.getAugmentation(Node1.class)
+            pccBuilder.setReportedLsp(initialNodeState.augmentation(Node1.class)
                     .getPathComputationClient().getReportedLsp());
         }
-        writeNode(pccBuilder, state, topologyAugment);
-        this.listenerState.init(session);
-        LOG.info("Session with {} attached to topology node {}", session.getRemoteAddress(), state.getNodeId());
-    }
-
-    private void writeNode(final PathComputationClientBuilder pccBuilder, final TopologyNodeState state,
-            final InstanceIdentifier<Node1> topologyAugment) {
-        final Node1 ta = new Node1Builder().setPathComputationClient(pccBuilder.build()).build();
-
-        final ReadWriteTransaction trans = state.rwTransaction();
-        trans.put(LogicalDatastoreType.OPERATIONAL, topologyAugment, ta);
-        LOG.trace("Peer data {} set to {}", topologyAugment, ta);
-
-        // All set, commit the modifications
-        Futures.addCallback(trans.submit(), new FutureCallback<Void>() {
-            @Override
-            public void onSuccess(final Void result) {
-                LOG.trace("Internal state for session {} updated successfully",
-                        AbstractTopologySessionListener.this.session);
-            }
-
-            @Override
-            public void onFailure(final Throwable t) {
-                LOG.error("Failed to update internal state for session {}, terminating it",
-                        AbstractTopologySessionListener.this.session, t);
-                AbstractTopologySessionListener.this.session.close(TerminationReason.UNKNOWN);
-            }
-        }, MoreExecutors.directExecutor());
+        state.storeNode(topologyAugment,
+                new Node1Builder().setPathComputationClient(pccBuilder.build()).build(), this.session);
+        this.listenerState.init(psession);
+        LOG.info("Session with {} attached to topology node {}", psession.getRemoteAddress(), state.getNodeId());
     }
 
-    void updatePccState(final PccSyncState pccSyncState) {
+    synchronized void updatePccState(final PccSyncState pccSyncState) {
         if (this.nodeState == null) {
             LOG.info("Server Session Manager is closed.");
             AbstractTopologySessionListener.this.session.close(TerminationReason.UNKNOWN);
             return;
         }
-        final MessageContext ctx = new MessageContext(this.nodeState.beginTransaction());
+        final MessageContext ctx = new MessageContext(this.nodeState.getChain().newWriteOnlyTransaction());
         updatePccNode(ctx, new PathComputationClientBuilder().setStateSync(pccSyncState).build());
         if (pccSyncState != PccSyncState.Synchronized) {
             this.synced.set(false);
             this.triggeredResyncInProcess = true;
         }
         // All set, commit the modifications
-        Futures.addCallback(ctx.trans.submit(), new FutureCallback<Void>() {
+        ctx.trans.commit().addCallback(new FutureCallback<CommitInfo>() {
             @Override
-            public void onSuccess(final Void result) {
-                LOG.trace("Internal state for session {} updated successfully", AbstractTopologySessionListener.this.session);
+            public void onSuccess(final CommitInfo result) {
+                LOG.trace("Pcc Internal state for session {} updated successfully",
+                        AbstractTopologySessionListener.this.session);
             }
 
             @Override
-            public void onFailure(final Throwable t) {
-                LOG.error("Failed to update internal state for session {}", AbstractTopologySessionListener.this.session, t);
+            public void onFailure(final Throwable throwable) {
+                LOG.error("Failed to update Pcc internal state for session {}",
+                        AbstractTopologySessionListener.this.session, throwable);
                 AbstractTopologySessionListener.this.session.close(TerminationReason.UNKNOWN);
             }
         }, MoreExecutors.directExecutor());
     }
 
-    boolean isTriggeredSyncInProcess() {
+    synchronized boolean isTriggeredSyncInProcess() {
         return this.triggeredResyncInProcess;
     }
 
     /**
      * Tear down the given PCEP session. It's OK to call this method even after the session
      * is already down. It always clear up the current session status.
-     *
-     * @param session
      */
     @GuardedBy("this")
-    private synchronized void tearDown(final PCEPSession session) {
+    @SuppressWarnings("checkstyle:IllegalCatch")
+    private synchronized void tearDown(final PCEPSession psession) {
 
-        requireNonNull(session);
-        this.serverSessionManager.releaseNodeState(this.nodeState, session, isLspDbPersisted());
-        if (this.nodeState != null) {
-            this.serverSessionManager.unbind(this.nodeState.getNodeId());
-            this.nodeState = null;
-        }
+        requireNonNull(psession);
+        this.serverSessionManager.releaseNodeState(this.nodeState, psession, isLspDbPersisted());
+        clearNodeState();
 
         try {
             if (this.session != null) {
                 this.session.close();
             }
-            session.close();
+            psession.close();
         } catch (final Exception e) {
-            LOG.error("Session {} cannot be closed.", session, e);
+            LOG.error("Session {} cannot be closed.", psession, e);
         }
         this.session = null;
         this.syncOptimization = null;
@@ -278,57 +259,76 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
     }
 
     @Override
-    public final synchronized void onSessionDown(final PCEPSession session, final Exception e) {
-        LOG.warn("Session {} went down unexpectedly", session, e);
-        tearDown(session);
+    public final synchronized void onSessionDown(final PCEPSession psession, final Exception exception) {
+        LOG.warn("Session {} went down unexpectedly", psession, exception);
+        tearDown(psession);
     }
 
     @Override
-    public final synchronized void onSessionTerminated(final PCEPSession session, final PCEPTerminationReason reason) {
-        LOG.info("Session {} terminated by peer with reason {}", session, reason);
-        tearDown(session);
+    public final synchronized void onSessionTerminated(final PCEPSession psession, final PCEPTerminationReason reason) {
+        LOG.info("Session {} terminated by peer with reason {}", psession, reason);
+        tearDown(psession);
     }
 
     @Override
-    public final synchronized void onMessage(final PCEPSession session, final Message message) {
+    public final synchronized void onMessage(final PCEPSession psession, final Message message) {
         if (this.nodeState == null) {
-            LOG.warn("Topology node state is null. Unhandled message {} on session {}", message, session);
-            session.close(TerminationReason.UNKNOWN);
+            LOG.warn("Topology node state is null. Unhandled message {} on session {}", message, psession);
+            psession.close(TerminationReason.UNKNOWN);
             return;
         }
-        final MessageContext ctx = new MessageContext(this.nodeState.beginTransaction());
+        final MessageContext ctx = new MessageContext(this.nodeState.getChain().newWriteOnlyTransaction());
 
         if (onMessage(ctx, message)) {
-            LOG.warn("Unhandled message {} on session {}", message, session);
+            LOG.warn("Unhandled message {} on session {}", message, psession);
             //cancel not supported, submit empty transaction
-            ctx.trans.submit();
+            ctx.trans.commit().addCallback(new FutureCallback<CommitInfo>() {
+                @Override
+                public void onSuccess(final CommitInfo result) {
+                    LOG.trace("Successful commit");
+                }
+
+                @Override
+                public void onFailure(final Throwable trw) {
+                    LOG.error("Failed commit", trw);
+                }
+            }, MoreExecutors.directExecutor());
             return;
         }
 
-        Futures.addCallback(ctx.trans.submit(), new FutureCallback<Void>() {
+        ctx.trans.commit().addCallback(new FutureCallback<CommitInfo>() {
             @Override
-            public void onSuccess(final Void result) {
-                LOG.trace("Internal state for session {} updated successfully", session);
+            public void onSuccess(final CommitInfo result) {
+                LOG.trace("Internal state for session {} updated successfully", psession);
                 ctx.notifyRequests();
+
             }
 
             @Override
-            public void onFailure(final Throwable t) {
-                LOG.error("Failed to update internal state for session {}, closing it", session, t);
+            public void onFailure(final Throwable throwable) {
+                LOG.error("Failed to update internal state for session {}, closing it", psession, throwable);
                 ctx.notifyRequests();
-                session.close(TerminationReason.UNKNOWN);
+                psession.close(TerminationReason.UNKNOWN);
             }
         }, MoreExecutors.directExecutor());
     }
 
     @Override
-    public void close() {
+    public synchronized void close() {
+        clearNodeState();
         if (this.session != null) {
             LOG.info("Closing session {}", session);
             this.session.close(TerminationReason.UNKNOWN);
         }
     }
 
+    private synchronized void clearNodeState() {
+        if (this.nodeState != null) {
+            this.serverSessionManager.unbind(this.nodeState.getNodeId());
+            this.nodeState = null;
+        }
+    }
+
     final synchronized PCEPRequest removeRequest(final S id) {
         final PCEPRequest ret = this.requests.remove(id);
         if (ret != null) {
@@ -382,7 +382,7 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
     }
 
     /**
-     * Update an LSP in the data store
+     * Update an LSP in the data store.
      *
      * @param ctx       Message context
      * @param id        Revision-specific LSP identifier
@@ -414,14 +414,14 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
         if (previous != null) {
             final List<Path> updatedPaths = makeBeforeBreak(rlb, previous, name, remove);
             // if all paths or the last path were deleted, delete whole tunnel
-            if (updatedPaths == null || updatedPaths.isEmpty()) {
+            if (updatedPaths.isEmpty()) {
                 LOG.debug("All paths were removed, removing LSP with {}.", id);
                 removeLsp(ctx, id);
                 return;
             }
             rlb.setPath(updatedPaths);
         }
-        rlb.setKey(new ReportedLspKey(name));
+        rlb.withKey(new ReportedLspKey(name));
         rlb.setName(name);
 
         // If this is an unsolicited update. We need to make sure we retain the metadata already present
@@ -432,16 +432,18 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
         }
 
         final ReportedLsp rl = rlb.build();
-        ctx.trans.put(LogicalDatastoreType.OPERATIONAL, this.pccIdentifier.child(ReportedLsp.class, rlb.getKey()), rl);
+        ctx.trans.put(LogicalDatastoreType.OPERATIONAL, this.pccIdentifier.child(ReportedLsp.class, rlb.key()), rl);
         LOG.debug("LSP {} updated to MD-SAL", name);
 
         this.lspData.put(name, rl);
     }
 
-    private List<Path> makeBeforeBreak(final ReportedLspBuilder rlb, final ReportedLsp previous, final String name, final boolean remove) {
+    private static List<Path> makeBeforeBreak(final ReportedLspBuilder rlb, final ReportedLsp previous,
+            final String name, final boolean remove) {
         // just one path should be reported
         Preconditions.checkState(rlb.getPath().size() == 1);
-        final org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.rsvp.rev150820.LspId reportedLspId = rlb.getPath().get(0).getLspId();
+        final org.opendaylight.yang.gen.v1.urn.opendaylight.params.xml.ns.yang.rsvp.rev150820.LspId reportedLspId =
+                rlb.getPath().get(0).getLspId();
         final List<Path> updatedPaths;
         //lspId = 0 and remove = false -> tunnel is down, still exists but no path is signaled
         //remove existing tunnel's paths now, as explicit path remove will not come
@@ -524,6 +526,7 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
         this.lspData.remove(name);
     }
 
+    @SuppressWarnings("checkstyle:OverloadMethodsDeclarationOrder")
     protected abstract void onSessionUp(PCEPSession session, PathComputationClientBuilder pccBuilder);
 
     /**
@@ -533,6 +536,7 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
      * @param message Protocol message
      * @return True if the message type is not handle.
      */
+    @SuppressWarnings("checkstyle:OverloadMethodsDeclarationOrder")
     protected abstract boolean onMessage(MessageContext ctx, Message message);
 
     final String lookupLspName(final L id) {
@@ -547,18 +551,18 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
      * @param id InstanceIdentifier of the node
      * @return null if the node does not exists, or operational data
      */
-    final synchronized <T extends DataObject> ListenableFuture<Optional<T>>
-    readOperationalData(final InstanceIdentifier<T> id) {
+    final synchronized <T extends DataObject> FluentFuture<Optional<T>>
+        readOperationalData(final InstanceIdentifier<T> id) {
         if (this.nodeState == null) {
             return null;
         }
         return this.nodeState.readOperationalData(id);
     }
 
-    protected abstract Object validateReportedLsp(final Optional<ReportedLsp> rep, final LspId input);
+    protected abstract Object validateReportedLsp(Optional<ReportedLsp> rep, LspId input);
 
-    protected abstract void loadLspData(final Node node, final Map<String, ReportedLsp> lspData,
-            final Map<L, String> lsps, final boolean incrementalSynchro);
+    protected abstract void loadLspData(Node node, Map<String, ReportedLsp> lspData, Map<L, String> lsps,
+            boolean incrementalSynchro);
 
     final boolean isLspDbPersisted() {
         return this.syncOptimization != null && this.syncOptimization.isSyncAvoidanceEnabled();
@@ -570,41 +574,40 @@ public abstract class AbstractTopologySessionListener<S, L> implements TopologyS
 
     /**
      * Is Incremental synchronization if LSP-DB-VERSION are included,
-     * LSP-DB-VERSION TLV values doesnt match, and  LSP-SYNC-CAPABILITY is enabled
-     *
-     * @return
+     * LSP-DB-VERSION TLV values doesnt match, and  LSP-SYNC-CAPABILITY is enabled.
      */
-    final boolean isIncrementalSynchro() {
-        return this.syncOptimization != null &&
-                this.syncOptimization.isSyncAvoidanceEnabled() &&
-                this.syncOptimization.isDeltaSyncEnabled();
+    final synchronized boolean isIncrementalSynchro() {
+        return this.syncOptimization != null && this.syncOptimization.isSyncAvoidanceEnabled()
+                && this.syncOptimization.isDeltaSyncEnabled();
     }
 
-    final boolean isTriggeredInitialSynchro() {
-        return this.syncOptimization != null &&
-                this.syncOptimization.isTriggeredInitSyncEnabled();
+    final synchronized boolean isTriggeredInitialSynchro() {
+        return this.syncOptimization != null && this.syncOptimization.isTriggeredInitSyncEnabled();
     }
 
-    final boolean isTriggeredReSyncEnabled() {
-        return this.syncOptimization != null &&
-                this.syncOptimization.isTriggeredReSyncEnabled();
+    final synchronized boolean isTriggeredReSyncEnabled() {
+        return this.syncOptimization != null && this.syncOptimization.isTriggeredReSyncEnabled();
     }
 
-    protected final boolean isSynchronized() {
-        return this.syncOptimization != null &&
-                this.syncOptimization.doesLspDbMatch();
+    protected final synchronized boolean isSynchronized() {
+        return this.syncOptimization != null && this.syncOptimization.doesLspDbMatch();
     }
 
     @Override
-    public synchronized int getDelegatedLspsCount() {
-        return Math.toIntExact(this.lspData.values().stream()
-                .map(ReportedLsp::getPath).filter(Objects::nonNull).filter(pathList -> !pathList.isEmpty())
-                // pick the first path, as delegate status should be same in each path
-                .map(pathList -> pathList.get(0))
-                .map(path -> path.getAugmentation(Path1.class)).filter(Objects::nonNull)
-                .map(LspObject::getLsp).filter(Objects::nonNull)
-                .filter(Lsp::isDelegate)
-                .count());
+    public int getDelegatedLspsCount() {
+        final Stream<ReportedLsp> stream;
+        synchronized (this) {
+            stream = ImmutableList.copyOf(this.lspData.values()).stream();
+        }
+
+        return Math.toIntExact(stream
+            .map(ReportedLsp::getPath).filter(pathList -> pathList != null && !pathList.isEmpty())
+            // pick the first path, as delegate status should be same in each path
+            .map(pathList -> pathList.get(0))
+            .map(path -> path.augmentation(Path1.class)).filter(Objects::nonNull)
+            .map(LspObject::getLsp).filter(Objects::nonNull)
+            .filter(Lsp::isDelegate)
+            .count());
     }
 
     @Override