Optimal TTL value for Module Sync WD
[cps.git] / cps-ncmp-service / src / main / java / org / onap / cps / ncmp / api / inventory / sync / ModuleSyncWatchdog.java
index c71f68f..8acaa0a 100644 (file)
@@ -1,6 +1,6 @@
 /*
  *  ============LICENSE_START=======================================================
- *  Copyright (C) 2022 Nordix Foundation
+ *  Copyright (C) 2022-2023 Nordix Foundation
  *  Modifications Copyright (C) 2022 Bell Canada
  *  ================================================================================
  *  Licensed under the Apache License, Version 2.0 (the "License");
 
 package org.onap.cps.ncmp.api.inventory.sync;
 
+import com.hazelcast.map.IMap;
+import java.util.Collection;
+import java.util.HashSet;
 import java.util.List;
-import java.util.concurrent.ConcurrentMap;
-import java.util.function.Consumer;
+import java.util.concurrent.BlockingQueue;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicInteger;
+import lombok.Getter;
 import lombok.RequiredArgsConstructor;
 import lombok.extern.slf4j.Slf4j;
+import org.onap.cps.ncmp.api.impl.config.embeddedcache.SynchronizationCacheConfig;
 import org.onap.cps.ncmp.api.impl.yangmodels.YangModelCmHandle;
-import org.onap.cps.ncmp.api.inventory.CmHandleState;
-import org.onap.cps.ncmp.api.inventory.CompositeState;
-import org.onap.cps.ncmp.api.inventory.DataStoreSyncState;
-import org.onap.cps.ncmp.api.inventory.InventoryPersistence;
-import org.onap.cps.ncmp.api.inventory.LockReasonCategory;
-import org.springframework.beans.factory.annotation.Value;
+import org.onap.cps.ncmp.api.inventory.sync.executor.AsyncTaskExecutor;
+import org.onap.cps.spi.model.DataNode;
 import org.springframework.scheduling.annotation.Scheduled;
-import org.springframework.stereotype.Component;
+import org.springframework.stereotype.Service;
 
 @Slf4j
 @RequiredArgsConstructor
-@Component
+@Service
 public class ModuleSyncWatchdog {
 
-    private final InventoryPersistence inventoryPersistence;
-
     private final SyncUtils syncUtils;
-
-    private final ModuleSyncService moduleSyncService;
-
-    @Value("${data-sync.cache.enabled:false}")
-    private boolean isGlobalDataSyncCacheEnabled;
-
-    private final ConcurrentMap<String, Boolean> moduleSyncSemaphoreMap;
+    private final BlockingQueue<DataNode> moduleSyncWorkQueue;
+    private final IMap<String, Object> moduleSyncStartedOnCmHandles;
+    private final ModuleSyncTasks moduleSyncTasks;
+    private final AsyncTaskExecutor asyncTaskExecutor;
+    private static final int MODULE_SYNC_BATCH_SIZE = 100;
+    private static final long PREVENT_CPU_BURN_WAIT_TIME_MILLIS = 10;
+    private static final String VALUE_FOR_HAZELCAST_IN_PROGRESS_MAP = "Started";
+    private static final long ASYNC_TASK_TIMEOUT_IN_MILLISECONDS = TimeUnit.MINUTES.toMillis(5);
+    @Getter
+    private AtomicInteger batchCounter = new AtomicInteger(1);
 
     /**
-     * Execute Cm Handle poll which changes the cm handle state from 'ADVISED' to 'READY'.
+     * Check DB for any cm handles in 'ADVISED' state.
+     * Queue and create batches to process them asynchronously.
+     * This method will only finish when there are no more 'ADVISED' cm handles in the DB.
+     * This method wil be triggered on a configurable interval
      */
-    @Scheduled(fixedDelayString = "${timers.advised-modules-sync.sleep-time-ms:30000}")
-    public void executeAdvisedCmHandlePoll() {
-        syncUtils.getAdvisedCmHandles().forEach(advisedCmHandle -> {
-            final String cmHandleId = advisedCmHandle.getId();
-            if (hasPushedIntoSemaphoreMap(cmHandleId)) {
-                log.debug("executing module sync on {}", cmHandleId);
-                final CompositeState compositeState = inventoryPersistence.getCmHandleState(cmHandleId);
-                try {
-                    moduleSyncService.deleteSchemaSetIfExists(advisedCmHandle);
-                    moduleSyncService.syncAndCreateSchemaSetAndAnchor(advisedCmHandle);
-                    setCompositeStateToReadyWithInitialDataStoreSyncState().accept(compositeState);
-                    updateModuleSyncSemaphoreMap(cmHandleId);
-                } catch (final Exception e) {
-                    setCompositeStateToLocked().accept(compositeState);
-                    syncUtils.updateLockReasonDetailsAndAttempts(compositeState,
-                            LockReasonCategory.LOCKED_MODULE_SYNC_FAILED, e.getMessage());
-                }
-                inventoryPersistence.saveCmHandleState(cmHandleId, compositeState);
-                log.debug("{} is now in {} state", cmHandleId, compositeState.getCmHandleState().name());
+    @Scheduled(fixedDelayString = "${ncmp.timers.advised-modules-sync.sleep-time-ms:5000}")
+    public void moduleSyncAdvisedCmHandles() {
+        log.info("Processing module sync watchdog waking up.");
+        populateWorkQueueIfNeeded();
+        final int asyncTaskParallelismLevel = asyncTaskExecutor.getAsyncTaskParallelismLevel();
+        while (!moduleSyncWorkQueue.isEmpty()) {
+            if (batchCounter.get() <= asyncTaskParallelismLevel) {
+                final Collection<DataNode> nextBatch = prepareNextBatch();
+                log.debug("Processing module sync batch of {}. {} batch(es) active.",
+                        nextBatch.size(), batchCounter.get());
+                asyncTaskExecutor.executeTask(() ->
+                                moduleSyncTasks.performModuleSync(nextBatch, batchCounter),
+                        ASYNC_TASK_TIMEOUT_IN_MILLISECONDS);
+                batchCounter.getAndIncrement();
             } else {
-                log.debug("{} already processed by another instance", cmHandleId);
+                preventBusyWait();
             }
-        });
-        log.debug("No Cm-Handles currently found in an ADVISED state");
+        }
     }
 
     /**
-     * Execute Cm Handle poll which changes the cm handle state from 'LOCKED' to 'ADVISED'.
+     * Find any failed (locked) cm handles and change state back to 'ADVISED'.
      */
-    @Scheduled(fixedDelayString = "${timers.locked-modules-sync.sleep-time-ms:300000}")
-    public void executeLockedCmHandlePoll() {
-        final List<YangModelCmHandle> lockedCmHandles = syncUtils.getModuleSyncFailedCmHandles();
-        for (final YangModelCmHandle lockedCmHandle : lockedCmHandles) {
-            final CompositeState compositeState = lockedCmHandle.getCompositeState();
-            final boolean isReadyForRetry = syncUtils.isReadyForRetry(compositeState);
-            if (isReadyForRetry) {
-                setCompositeStateToAdvisedAndRetainOldLockReasonDetails(compositeState);
-                log.debug("Locked cm handle {} is being re-synced", lockedCmHandle.getId());
-                inventoryPersistence.saveCmHandleState(lockedCmHandle.getId(), compositeState);
-            }
-        }
-    }
-
-    private Consumer<CompositeState> setCompositeStateToLocked() {
-        return compositeState -> {
-            compositeState.setCmHandleState(CmHandleState.LOCKED);
-            compositeState.setLastUpdateTimeNow();
-        };
+    @Scheduled(fixedDelayString = "${ncmp.timers.locked-modules-sync.sleep-time-ms:300000}")
+    public void resetPreviouslyFailedCmHandles() {
+        log.info("Processing module sync retry-watchdog waking up.");
+        final List<YangModelCmHandle> failedCmHandles = syncUtils.getModuleSyncFailedCmHandles();
+        log.info("Retrying {} cmHandles", failedCmHandles.size());
+        moduleSyncTasks.resetFailedCmHandles(failedCmHandles);
     }
 
-    private Consumer<CompositeState> setCompositeStateToReadyWithInitialDataStoreSyncState() {
-        return compositeState -> {
-            compositeState.setDataSyncEnabled(isGlobalDataSyncCacheEnabled);
-            compositeState.setCmHandleState(CmHandleState.READY);
-            final CompositeState.Operational operational = getDataStoreSyncState(compositeState.getDataSyncEnabled());
-            final CompositeState.DataStores dataStores = CompositeState.DataStores.builder()
-                    .operationalDataStore(operational)
-                    .build();
-            compositeState.setDataStores(dataStores);
-        };
-    }
-
-    private void setCompositeStateToAdvisedAndRetainOldLockReasonDetails(final CompositeState compositeState) {
-        compositeState.setCmHandleState(CmHandleState.ADVISED);
-        compositeState.setLastUpdateTimeNow();
-        final String oldLockReasonDetails = compositeState.getLockReason().getDetails();
-        final CompositeState.LockReason lockReason = CompositeState.LockReason.builder()
-                .details(oldLockReasonDetails).build();
-        compositeState.setLockReason(lockReason);
+    private void preventBusyWait() {
+        try {
+            log.info("Busy waiting now");
+            TimeUnit.MILLISECONDS.sleep(PREVENT_CPU_BURN_WAIT_TIME_MILLIS);
+        } catch (final InterruptedException e) {
+            Thread.currentThread().interrupt();
+        }
     }
 
-    private CompositeState.Operational getDataStoreSyncState(final boolean dataSyncEnabled) {
-        final DataStoreSyncState dataStoreSyncState = dataSyncEnabled
-                ? DataStoreSyncState.UNSYNCHRONIZED : DataStoreSyncState.NONE_REQUESTED;
-        return CompositeState.Operational.builder().dataStoreSyncState(dataStoreSyncState).build();
+    private void populateWorkQueueIfNeeded() {
+        if (moduleSyncWorkQueue.isEmpty()) {
+            final List<DataNode> advisedCmHandles = syncUtils.getAdvisedCmHandles();
+            log.info("Processing module sync fetched {} advised cm handles from DB", advisedCmHandles.size());
+            for (final DataNode advisedCmHandle : advisedCmHandles) {
+                if (!moduleSyncWorkQueue.offer(advisedCmHandle)) {
+                    log.warn("Unable to add cm handle {} to the work queue", advisedCmHandle.getLeaves().get("id"));
+                }
+            }
+            log.info("Work Queue Size : {}", moduleSyncWorkQueue.size());
+        }
     }
 
-    private void updateModuleSyncSemaphoreMap(final String cmHandleId) {
-        moduleSyncSemaphoreMap.replace(cmHandleId, true);
+    private Collection<DataNode> prepareNextBatch() {
+        final Collection<DataNode> nextBatchCandidates = new HashSet<>(MODULE_SYNC_BATCH_SIZE);
+        final Collection<DataNode> nextBatch = new HashSet<>(MODULE_SYNC_BATCH_SIZE);
+        moduleSyncWorkQueue.drainTo(nextBatchCandidates, MODULE_SYNC_BATCH_SIZE);
+        log.debug("nextBatchCandidates size : {}", nextBatchCandidates.size());
+        for (final DataNode batchCandidate : nextBatchCandidates) {
+            final String cmHandleId = String.valueOf(batchCandidate.getLeaves().get("id"));
+            final boolean alreadyAddedToInProgressMap = VALUE_FOR_HAZELCAST_IN_PROGRESS_MAP.equals(
+                    moduleSyncStartedOnCmHandles.putIfAbsent(cmHandleId, VALUE_FOR_HAZELCAST_IN_PROGRESS_MAP,
+                            SynchronizationCacheConfig.MODULE_SYNC_STARTED_TTL_SECS, TimeUnit.SECONDS));
+            if (alreadyAddedToInProgressMap) {
+                log.debug("module sync for {} already in progress by other instance", cmHandleId);
+            } else {
+                nextBatch.add(batchCandidate);
+            }
+        }
+        log.debug("nextBatch size : {}", nextBatch.size());
+        return nextBatch;
     }
 
-    private boolean hasPushedIntoSemaphoreMap(final String cmHandleId) {
-        return moduleSyncSemaphoreMap.putIfAbsent(cmHandleId, false) == null;
-    }
 }