diff options
Diffstat (limited to 'cps-ncmp-service/src')
11 files changed, 264 insertions, 113 deletions
diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/BatchRecordFilterStrategy.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/DataOperationRecordFilterStrategy.java index b343d70a7a..9e2b66a2c1 100644 --- a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/BatchRecordFilterStrategy.java +++ b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/DataOperationRecordFilterStrategy.java @@ -22,17 +22,17 @@ package org.onap.cps.ncmp.api.impl.async; import org.apache.commons.lang3.SerializationUtils; import org.apache.kafka.common.header.Header; -import org.onap.cps.ncmp.events.async.BatchDataResponseEventV1; +import org.onap.cps.ncmp.events.async1_0_0.DataOperationEvent; import org.springframework.context.annotation.Bean; import org.springframework.context.annotation.Configuration; import org.springframework.kafka.listener.adapter.RecordFilterStrategy; /** - * Batch Record filter strategy, which helps to filter the consumer records. + * Data operation record filter strategy, which helps to filter the consumer records. * */ @Configuration -public class BatchRecordFilterStrategy { +public class DataOperationRecordFilterStrategy { /** * Filtering the consumer records based on the eventType header, It @@ -41,7 +41,7 @@ public class BatchRecordFilterStrategy { * @return boolean value. */ @Bean - public RecordFilterStrategy<String, BatchDataResponseEventV1> filterBatchDataResponseEvent() { + public RecordFilterStrategy<String, DataOperationEvent> includeDataOperationEventsOnly() { return consumedRecord -> { final Header eventTypeHeader = consumedRecord.headers().lastHeader("eventType"); if (eventTypeHeader == null) { @@ -49,7 +49,7 @@ public class BatchRecordFilterStrategy { } final String eventTypeHeaderValue = SerializationUtils.deserialize(eventTypeHeader.value()); return !(eventTypeHeaderValue != null - && eventTypeHeaderValue.startsWith("org.onap.cps.ncmp.events.async.BatchDataResponseEvent")); + && eventTypeHeaderValue.contains("DataOperationEvent")); }; } } diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/NcmpAsyncBatchEventConsumer.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/NcmpAsyncDataOperationEventConsumer.java index 2a332d0037..995a4d5a67 100644 --- a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/NcmpAsyncBatchEventConsumer.java +++ b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/async/NcmpAsyncDataOperationEventConsumer.java @@ -25,40 +25,41 @@ import lombok.extern.slf4j.Slf4j; import org.apache.commons.lang3.SerializationUtils; import org.apache.kafka.clients.consumer.ConsumerRecord; import org.onap.cps.ncmp.api.impl.events.EventsPublisher; -import org.onap.cps.ncmp.events.async.BatchDataResponseEventV1; +import org.onap.cps.ncmp.events.async1_0_0.DataOperationEvent; import org.springframework.boot.autoconfigure.condition.ConditionalOnProperty; import org.springframework.kafka.annotation.KafkaListener; import org.springframework.stereotype.Component; /** - * Listener for cps-ncmp async batch events. + * Listener for cps-ncmp async data operation events. */ @Component @Slf4j @RequiredArgsConstructor @ConditionalOnProperty(name = "notification.enabled", havingValue = "true", matchIfMissing = true) -public class NcmpAsyncBatchEventConsumer { +public class NcmpAsyncDataOperationEventConsumer { - private final EventsPublisher<BatchDataResponseEventV1> eventsPublisher; + private final EventsPublisher<DataOperationEvent> eventsPublisher; /** - * Consume the BatchDataResponseEvent published by producer to topic 'async-m2m.topic' + * Consume the DataOperationResponseEvent published by producer to topic 'async-m2m.topic' * and publish the same to the client specified topic. * - * @param batchEventConsumerRecord consuming event as a ConsumerRecord. + * @param dataOperationEventConsumerRecord consuming event as a ConsumerRecord. */ @KafkaListener( topics = "${app.ncmp.async-m2m.topic}", - filter = "filterBatchDataResponseEvent", - groupId = "ncmp-batch-event-group", - properties = {"spring.json.value.default.type=org.onap.cps.ncmp.events.async.BatchDataResponseEventV1"}) - public void consumeAndPublish(final ConsumerRecord<String, BatchDataResponseEventV1> batchEventConsumerRecord) { - log.info("Consuming event payload {} ...", batchEventConsumerRecord.value()); + filter = "includeDataOperationEventsOnly", + groupId = "ncmp-data-operation-event-group", + properties = {"spring.json.value.default.type=org.onap.cps.ncmp.events.async1_0_0.DataOperationEvent"}) + public void consumeAndPublish(final ConsumerRecord<String, DataOperationEvent> + dataOperationEventConsumerRecord) { + log.info("Consuming event payload {} ...", dataOperationEventConsumerRecord.value()); final String eventTarget = SerializationUtils - .deserialize(batchEventConsumerRecord.headers().lastHeader("eventTarget").value()); + .deserialize(dataOperationEventConsumerRecord.headers().lastHeader("eventTarget").value()); final String eventId = SerializationUtils - .deserialize(batchEventConsumerRecord.headers().lastHeader("eventId").value()); - eventsPublisher.publishEvent(eventTarget, eventId, batchEventConsumerRecord.headers(), - batchEventConsumerRecord.value()); + .deserialize(dataOperationEventConsumerRecord.headers().lastHeader("eventId").value()); + eventsPublisher.publishEvent(eventTarget, eventId, dataOperationEventConsumerRecord.headers(), + dataOperationEventConsumerRecord.value()); } } diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/events/avcsubscription/SubscriptionEventResponseOutcome.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/events/avcsubscription/SubscriptionEventResponseOutcome.java index a74682571b..1bfc4ab28b 100644 --- a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/events/avcsubscription/SubscriptionEventResponseOutcome.java +++ b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/events/avcsubscription/SubscriptionEventResponseOutcome.java @@ -22,8 +22,6 @@ package org.onap.cps.ncmp.api.impl.events.avcsubscription; import java.io.Serializable; import java.util.Collection; -import java.util.HashMap; -import java.util.Iterator; import java.util.List; import java.util.Map; import lombok.RequiredArgsConstructor; @@ -104,29 +102,12 @@ public class SubscriptionEventResponseOutcome { private SubscriptionEventResponse toSubscriptionEventResponse( final List<Collection<Serializable>> cmHandleIdToStatus, final String subscriptionClientId, final String subscriptionName) { - final Map<String, SubscriptionStatus> cmHandleIdToStatusMap = new HashMap<>(); + final Map<String, SubscriptionStatus> cmHandleIdToStatusMap = + DataNodeHelper.getCmHandleIdToStatusMap(cmHandleIdToStatus); + final SubscriptionEventResponse subscriptionEventResponse = new SubscriptionEventResponse(); subscriptionEventResponse.setClientId(subscriptionClientId); subscriptionEventResponse.setSubscriptionName(subscriptionName); - - for (final Collection<Serializable> cmHandleToStatusBucket: cmHandleIdToStatus) { - final Iterator<Serializable> bucketIterator = cmHandleToStatusBucket.iterator(); - while (bucketIterator.hasNext()) { - final String item = (String) bucketIterator.next(); - if ("PENDING".equals(item)) { - cmHandleIdToStatusMap.put((String) bucketIterator.next(), - SubscriptionStatus.PENDING); - } - if ("REJECTED".equals(item)) { - cmHandleIdToStatusMap.put((String) bucketIterator.next(), - SubscriptionStatus.REJECTED); - } - if ("ACCEPTED".equals(item)) { - cmHandleIdToStatusMap.put((String) bucketIterator.next(), - SubscriptionStatus.ACCEPTED); - } - } - } subscriptionEventResponse.setCmHandleIdToStatus(cmHandleIdToStatusMap); return subscriptionEventResponse; diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistence.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistence.java index f240c4510d..27d4266566 100644 --- a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistence.java +++ b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistence.java @@ -39,4 +39,11 @@ public interface SubscriptionPersistence { * @return the DataNode as collection. */ Collection<DataNode> getDataNodesForSubscriptionEvent(); + + /** + * Get data nodes by xpath. + * + * @return the DataNode as collection. + */ + Collection<DataNode> getCmHandlesForSubscriptionEvent(String clientId, String subscriptionName); } diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceImpl.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceImpl.java index 9a063d6dfd..d2b1237a4d 100644 --- a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceImpl.java +++ b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceImpl.java @@ -22,11 +22,18 @@ package org.onap.cps.ncmp.api.impl.subscriptions; import static org.onap.cps.ncmp.api.impl.constants.DmiRegistryConstants.NO_TIMESTAMP; +import java.io.Serializable; +import java.util.Arrays; import java.util.Collection; +import java.util.HashMap; +import java.util.List; +import java.util.Map; import java.util.Optional; +import java.util.stream.Collectors; import lombok.RequiredArgsConstructor; import lombok.extern.slf4j.Slf4j; import org.onap.cps.api.CpsDataService; +import org.onap.cps.ncmp.api.impl.utils.DataNodeHelper; import org.onap.cps.ncmp.api.impl.yangmodels.YangModelSubscriptionEvent; import org.onap.cps.spi.FetchDescendantsOption; import org.onap.cps.spi.model.DataNode; @@ -41,35 +48,86 @@ public class SubscriptionPersistenceImpl implements SubscriptionPersistence { private static final String SUBSCRIPTION_DATASPACE_NAME = "NCMP-Admin"; private static final String SUBSCRIPTION_ANCHOR_NAME = "AVC-Subscriptions"; private static final String SUBSCRIPTION_REGISTRY_PARENT = "/subscription-registry"; - private final JsonObjectMapper jsonObjectMapper; private final CpsDataService cpsDataService; @Override public void saveSubscriptionEvent(final YangModelSubscriptionEvent yangModelSubscriptionEvent) { - final String subscriptionEventJsonData = - createSubscriptionEventJsonData(jsonObjectMapper.asJsonString(yangModelSubscriptionEvent)); + final String clientId = yangModelSubscriptionEvent.getClientId(); + final String subscriptionName = yangModelSubscriptionEvent.getSubscriptionName(); + final Collection<DataNode> dataNodes = cpsDataService.getDataNodes(SUBSCRIPTION_DATASPACE_NAME, SUBSCRIPTION_ANCHOR_NAME, SUBSCRIPTION_REGISTRY_PARENT, FetchDescendantsOption.INCLUDE_ALL_DESCENDANTS); + + if (isSubscriptionRegistryEmptyOrNonExist(dataNodes, clientId, subscriptionName)) { + saveSubscriptionEventYangModel(createSubscriptionEventJsonData( + jsonObjectMapper.asJsonString(yangModelSubscriptionEvent))); + } else { + findDeltaCmHandlesAddOrUpdateInDatabase(yangModelSubscriptionEvent, clientId, subscriptionName, dataNodes); + } + } + + private void findDeltaCmHandlesAddOrUpdateInDatabase(final YangModelSubscriptionEvent yangModelSubscriptionEvent, + final String clientId, final String subscriptionName, + final Collection<DataNode> dataNodes) { + final Map<String, SubscriptionStatus> cmHandleIdsFromYangModel = + extractCmHandleFromYangModelAsMap(yangModelSubscriptionEvent); + final Map<String, SubscriptionStatus> cmHandleIdsFromDatabase = + extractCmHandleFromDbAsMap(dataNodes); + + final Map<String, SubscriptionStatus> newCmHandles = + mapDifference(cmHandleIdsFromYangModel, cmHandleIdsFromDatabase); + traverseCmHandleList(newCmHandles, clientId, subscriptionName, true); + + final Map<String, SubscriptionStatus> existingCmHandles = + mapDifference(cmHandleIdsFromYangModel, newCmHandles); + traverseCmHandleList(existingCmHandles, clientId, subscriptionName, false); + } + + private boolean isSubscriptionRegistryEmptyOrNonExist(final Collection<DataNode> dataNodes, + final String clientId, final String subscriptionName) { final Optional<DataNode> dataNodeFirst = dataNodes.stream().findFirst(); - final boolean isCreateOperation = - dataNodeFirst.isPresent() && dataNodeFirst.get().getChildDataNodes().isEmpty(); - saveOrUpdateSubscriptionEventYangModel(subscriptionEventJsonData, isCreateOperation); + return ((dataNodeFirst.isPresent() && dataNodeFirst.get().getChildDataNodes().isEmpty()) + || getCmHandlesForSubscriptionEvent(clientId, subscriptionName).isEmpty()); + } + + private void traverseCmHandleList(final Map<String, SubscriptionStatus> cmHandleMap, + final String clientId, + final String subscriptionName, + final boolean isAddListElementOperation) { + final List<YangModelSubscriptionEvent.TargetCmHandle> cmHandleList = + targetCmHandlesAsList(cmHandleMap); + for (final YangModelSubscriptionEvent.TargetCmHandle targetCmHandle : cmHandleList) { + final String targetCmHandleAsJson = + createTargetCmHandleJsonData(jsonObjectMapper.asJsonString(targetCmHandle)); + addOrReplaceCmHandlePredicateListElement(targetCmHandleAsJson, clientId, subscriptionName, + isAddListElementOperation); + } } - private void saveOrUpdateSubscriptionEventYangModel(final String subscriptionEventJsonData, - final boolean isCreateOperation) { - if (isCreateOperation) { - log.info("SubscriptionEventJsonData to be saved into DB {}", subscriptionEventJsonData); - cpsDataService.saveListElements(SUBSCRIPTION_DATASPACE_NAME, SUBSCRIPTION_ANCHOR_NAME, - SUBSCRIPTION_REGISTRY_PARENT, subscriptionEventJsonData, NO_TIMESTAMP); + private void addOrReplaceCmHandlePredicateListElement(final String targetCmHandleAsJson, + final String clientId, + final String subscriptionName, + final boolean isAddListElementOperation) { + if (isAddListElementOperation) { + log.info("targetCmHandleAsJson to be added into DB {}", targetCmHandleAsJson); + cpsDataService.saveListElements(SUBSCRIPTION_DATASPACE_NAME, + SUBSCRIPTION_ANCHOR_NAME, createCmHandleXpathPredicates(clientId, subscriptionName), + targetCmHandleAsJson, NO_TIMESTAMP); } else { - log.info("SubscriptionEventJsonData to be updated into DB {}", subscriptionEventJsonData); - cpsDataService.updateDataNodeAndDescendants(SUBSCRIPTION_DATASPACE_NAME, SUBSCRIPTION_ANCHOR_NAME, - SUBSCRIPTION_REGISTRY_PARENT, subscriptionEventJsonData, NO_TIMESTAMP); + log.info("targetCmHandleAsJson to be updated into DB {}", targetCmHandleAsJson); + cpsDataService.updateNodeLeaves(SUBSCRIPTION_DATASPACE_NAME, + SUBSCRIPTION_ANCHOR_NAME, createCmHandleXpathPredicates(clientId, subscriptionName), + targetCmHandleAsJson, NO_TIMESTAMP); } } + private void saveSubscriptionEventYangModel(final String subscriptionEventJsonData) { + log.info("SubscriptionEventJsonData to be saved into DB {}", subscriptionEventJsonData); + cpsDataService.saveListElements(SUBSCRIPTION_DATASPACE_NAME, SUBSCRIPTION_ANCHOR_NAME, + SUBSCRIPTION_REGISTRY_PARENT, subscriptionEventJsonData, NO_TIMESTAMP); + } + @Override public Collection<DataNode> getDataNodesForSubscriptionEvent() { return cpsDataService.getDataNodes(SUBSCRIPTION_DATASPACE_NAME, @@ -77,7 +135,58 @@ public class SubscriptionPersistenceImpl implements SubscriptionPersistence { FetchDescendantsOption.INCLUDE_ALL_DESCENDANTS); } + @Override + public Collection<DataNode> getCmHandlesForSubscriptionEvent(final String clientId, final String subscriptionName) { + return cpsDataService.getDataNodesForMultipleXpaths(SUBSCRIPTION_DATASPACE_NAME, + SUBSCRIPTION_ANCHOR_NAME, Arrays.asList(createCmHandleXpath(clientId, subscriptionName)), + FetchDescendantsOption.INCLUDE_ALL_DESCENDANTS); + } + + private static Map<String, SubscriptionStatus> extractCmHandleFromDbAsMap(final Collection<DataNode> dataNodes) { + final List<Map<String, Serializable>> dataNodeLeaves = DataNodeHelper.getDataNodeLeaves(dataNodes); + final List<Collection<Serializable>> cmHandleIdToStatus = DataNodeHelper.getCmHandleIdToStatus(dataNodeLeaves); + return DataNodeHelper.getCmHandleIdToStatusMap(cmHandleIdToStatus); + } + + private static Map<String, SubscriptionStatus> extractCmHandleFromYangModelAsMap( + final YangModelSubscriptionEvent yangModelSubscriptionEvent) { + return yangModelSubscriptionEvent.getPredicates().getTargetCmHandles() + .stream().collect(Collectors.toMap( + YangModelSubscriptionEvent.TargetCmHandle::getCmHandleId, + YangModelSubscriptionEvent.TargetCmHandle::getStatus)); + } + + private static List<YangModelSubscriptionEvent.TargetCmHandle> targetCmHandlesAsList( + final Map<String, SubscriptionStatus> newCmHandles) { + return newCmHandles.entrySet().stream().map(entry -> + new YangModelSubscriptionEvent.TargetCmHandle(entry.getKey(), + entry.getValue())).collect(Collectors.toList()); + } + private static String createSubscriptionEventJsonData(final String yangModelSubscriptionAsJson) { return "{\"subscription\":[" + yangModelSubscriptionAsJson + "]}"; } + + private static String createTargetCmHandleJsonData(final String targetCmHandleAsJson) { + return "{\"targetCmHandles\":[" + targetCmHandleAsJson + "]}"; + } + + private static String createCmHandleXpathPredicates(final String clientId, final String subscriptionName) { + return "/subscription-registry/subscription[@clientID='" + clientId + + "' and @subscriptionName='" + subscriptionName + "']/predicates"; + } + + private static String createCmHandleXpath(final String clientId, final String subscriptionName) { + return "/subscription-registry/subscription[@clientID='" + clientId + + "' and @subscriptionName='" + subscriptionName + "']"; + } + + private static <K, V> Map<K, V> mapDifference(final Map<? extends K, ? extends V> left, + final Map<? extends K, ? extends V> right) { + final Map<K, V> difference = new HashMap<>(); + difference.putAll(left); + difference.putAll(right); + difference.entrySet().removeAll(right.entrySet()); + return difference; + } } diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionStatus.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionStatus.java index 0b4f91fac3..ce3b88ba03 100644 --- a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionStatus.java +++ b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionStatus.java @@ -20,8 +20,36 @@ package org.onap.cps.ncmp.api.impl.subscriptions; +import java.io.Serializable; +import java.util.Iterator; +import java.util.Map; + public enum SubscriptionStatus { ACCEPTED, REJECTED, - PENDING + PENDING; + + + /** + * Populates a map with a key of cm handle id and a value of subscription status. + * + * @param resultMap the map is being populated + * @param bucketIterator to iterate over the collection + */ + public static void populateCmHandleToSubscriptionStatusMap(final Map<String, SubscriptionStatus> resultMap, + final Iterator<Serializable> bucketIterator) { + final String item = (String) bucketIterator.next(); + if ("PENDING".equals(item)) { + resultMap.put((String) bucketIterator.next(), + SubscriptionStatus.PENDING); + } + if ("REJECTED".equals(item)) { + resultMap.put((String) bucketIterator.next(), + SubscriptionStatus.REJECTED); + } + if ("ACCEPTED".equals(item)) { + resultMap.put((String) bucketIterator.next(), + SubscriptionStatus.ACCEPTED); + } + } } diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/utils/DataNodeHelper.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/utils/DataNodeHelper.java index 1648ac4fbb..8d44592ae2 100644 --- a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/utils/DataNodeHelper.java +++ b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/utils/DataNodeHelper.java @@ -22,12 +22,15 @@ package org.onap.cps.ncmp.api.impl.utils; import java.io.Serializable; import java.util.Collection; +import java.util.HashMap; +import java.util.Iterator; import java.util.List; import java.util.Map; import java.util.stream.Collectors; import java.util.stream.Stream; import lombok.AccessLevel; import lombok.NoArgsConstructor; +import org.onap.cps.ncmp.api.impl.subscriptions.SubscriptionStatus; import org.onap.cps.spi.model.DataNode; @NoArgsConstructor(access = AccessLevel.PRIVATE) @@ -72,4 +75,22 @@ public class DataNodeHelper { || col.contains("REJECTED")) .collect(Collectors.toList()); } + + /** + * The cm handle and status is returned as a map. + * + * @param cmHandleIdToStatus as a list of collection + * @return a map of cm handle id to status + */ + public static Map<String, SubscriptionStatus> getCmHandleIdToStatusMap( + final List<Collection<Serializable>> cmHandleIdToStatus) { + final Map<String, SubscriptionStatus> resultMap = new HashMap<>(); + for (final Collection<Serializable> cmHandleToStatusBucket: cmHandleIdToStatus) { + final Iterator<Serializable> bucketIterator = cmHandleToStatusBucket.iterator(); + while (bucketIterator.hasNext()) { + SubscriptionStatus.populateCmHandleToSubscriptionStatusMap(resultMap, bucketIterator); + } + } + return resultMap; + } } diff --git a/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/async/NcmpAsyncBatchEventConsumerSpec.groovy b/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/async/NcmpAsyncDataOperationEventConsumerSpec.groovy index 02071cd8cf..d9b9ce6db0 100644 --- a/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/async/NcmpAsyncBatchEventConsumerSpec.groovy +++ b/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/async/NcmpAsyncDataOperationEventConsumerSpec.groovy @@ -28,7 +28,7 @@ import org.apache.kafka.common.header.internals.RecordHeader import org.apache.kafka.common.serialization.StringDeserializer import org.onap.cps.ncmp.api.impl.events.EventsPublisher import org.onap.cps.ncmp.api.kafka.MessagingBaseSpec -import org.onap.cps.ncmp.events.async.BatchDataResponseEventV1 +import org.onap.cps.ncmp.events.async1_0_0.DataOperationEvent import org.onap.cps.ncmp.utils.TestUtils import org.onap.cps.utils.JsonObjectMapper import org.spockframework.spring.SpringBean @@ -37,43 +37,41 @@ import org.springframework.boot.test.context.SpringBootTest import org.springframework.kafka.listener.adapter.RecordFilterStrategy import org.springframework.test.annotation.DirtiesContext import org.testcontainers.spock.Testcontainers - import java.time.Duration -@SpringBootTest(classes = [EventsPublisher, NcmpAsyncBatchEventConsumer, BatchRecordFilterStrategy,JsonObjectMapper, - ObjectMapper]) +@SpringBootTest(classes = [EventsPublisher, NcmpAsyncDataOperationEventConsumer, DataOperationRecordFilterStrategy,JsonObjectMapper, ObjectMapper]) @Testcontainers @DirtiesContext -class NcmpAsyncBatchEventConsumerSpec extends MessagingBaseSpec { +class NcmpAsyncDataOperationEventConsumerSpec extends MessagingBaseSpec { @SpringBean - EventsPublisher asyncBatchEventPublisher = new EventsPublisher<BatchDataResponseEventV1>(legacyEventKafkaTemplate, cloudEventKafkaTemplate) + EventsPublisher asyncDataOperationEventPublisher = new EventsPublisher<DataOperationEvent>(legacyEventKafkaTemplate, cloudEventKafkaTemplate) @SpringBean - NcmpAsyncBatchEventConsumer asyncBatchEventConsumer = new NcmpAsyncBatchEventConsumer(asyncBatchEventPublisher) + NcmpAsyncDataOperationEventConsumer asyncDataOperationEventConsumer = new NcmpAsyncDataOperationEventConsumer(asyncDataOperationEventPublisher) @Autowired JsonObjectMapper jsonObjectMapper @Autowired - RecordFilterStrategy<String, BatchDataResponseEventV1> recordFilterStrategy + RecordFilterStrategy<String, DataOperationEvent> recordFilterStrategy def legacyEventKafkaConsumer = new KafkaConsumer<>(eventConsumerConfigProperties('test', StringDeserializer)) def static clientTopic = 'client-topic' - def static batchEventType = 'org.onap.cps.ncmp.events.async.BatchDataResponseEventV1' + def static dataOperationType = 'org.onap.cps.ncmp.events.async1_0_0.DataOperationEvent' def 'Consume and publish event to client specified topic'() { given: 'consumer subscribing to client topic' legacyEventKafkaConsumer.subscribe([clientTopic]) - and: 'consumer record for batch event' - def consumerRecordIn = createConsumerRecord(batchEventType) - when: 'the batch event is consumed and published to client specified topic' - asyncBatchEventConsumer.consumeAndPublish(consumerRecordIn) + and: 'consumer record for data operation event' + def consumerRecordIn = createConsumerRecord(dataOperationType) + when: 'the data operation event is consumed and published to client specified topic' + asyncDataOperationEventConsumer.consumeAndPublish(consumerRecordIn) and: 'the client specified topic is polled' def consumerRecordOut = legacyEventKafkaConsumer.poll(Duration.ofMillis(1500))[0] then: 'verifying consumed event operationID is same as published event operationID' - def operationIdIn = consumerRecordIn.value.event.batchResponses[0].operationId - def operationIdOut = jsonObjectMapper.convertJsonString((String)consumerRecordOut.value(), BatchDataResponseEventV1.class).event.batchResponses[0].operationId + def operationIdIn = consumerRecordIn.value.data.responses[0].operationId + def operationIdOut = jsonObjectMapper.convertJsonString((String)consumerRecordOut.value(), DataOperationEvent.class).data.responses[0].operationId assert operationIdIn == operationIdOut } @@ -85,14 +83,14 @@ class NcmpAsyncBatchEventConsumerSpec extends MessagingBaseSpec { then: 'the event is #description' assert result == expectedResult where: 'filter the event based on the eventType #eventType' - description | eventType || expectedResult - 'not filtered(the consumer will see the event)' | batchEventType || false - 'filtered(the consumer will not see the event)' | 'wrongType' || true + description | eventType || expectedResult + 'not filtered(the consumer will see the event)' | dataOperationType || false + 'filtered(the consumer will not see the event)' | 'wrongType' || true } def createConsumerRecord(eventTypeAsString) { - def jsonData = TestUtils.getResourceFileContent('batchDataEvent.json') - def testEventSent = jsonObjectMapper.convertJsonString(jsonData, BatchDataResponseEventV1.class) + def jsonData = TestUtils.getResourceFileContent('dataOperationEvent.json') + def testEventSent = jsonObjectMapper.convertJsonString(jsonData, DataOperationEvent.class) def eventTarget = SerializationUtils.serialize(clientTopic) def eventType = SerializationUtils.serialize(eventTypeAsString) def eventId = SerializationUtils.serialize('12345') diff --git a/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceSpec.groovy b/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceSpec.groovy index a372abe6ff..ec54e8917a 100644 --- a/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceSpec.groovy +++ b/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/subscriptions/SubscriptionPersistenceSpec.groovy @@ -34,6 +34,7 @@ class SubscriptionPersistenceSpec extends Specification { private static final String SUBSCRIPTION_DATASPACE_NAME = "NCMP-Admin"; private static final String SUBSCRIPTION_ANCHOR_NAME = "AVC-Subscriptions"; private static final String SUBSCRIPTION_REGISTRY_PARENT = "/subscription-registry"; + private static final String SUBSCRIPTION_REGISTRY_PREDICATES_XPATH = "/subscription-registry/subscription[@clientID='some-client-id' and @subscriptionName='some-subscription-name']/predicates"; def jsonObjectMapper = new JsonObjectMapper(new ObjectMapper()) def mockCpsDataService = Mock(CpsDataService) @@ -45,11 +46,11 @@ class SubscriptionPersistenceSpec extends Specification { def yangModelSubscriptionEvent = new YangModelSubscriptionEvent(clientId: 'some-client-id', subscriptionName: 'some-subscription-name', tagged: true, topic: 'some-topic', predicates: predicates) - def 'save a subscription event' () { - given: 'a data node that does not exist in db' + def 'save a subscription event as yang model into db for the #scenarios' () { + given: 'a blank data node that exist in db' def blankDataNode = new DataNodeBuilder().withDataspace('NCMP-Admin') .withAnchor('AVC-Subscriptions').withXpath('/subscription-registry').build() - and: 'cps data service return non existing data node' + and: 'cps data service return an empty data node' mockCpsDataService.getDataNodes(*_) >> [blankDataNode] when: 'the yangModelSubscriptionEvent is saved into db' objectUnderTest.saveSubscriptionEvent(yangModelSubscriptionEvent) @@ -63,24 +64,28 @@ class SubscriptionPersistenceSpec extends Specification { NO_TIMESTAMP) } - def 'update a subscription event' () { - given: 'a data node exist in db' + def 'add or replace cm handle list element into db' () { + given: 'a data node with child node exist in db' + def leaves1 = [status:'PENDING', cmHandleId:'cmhandle1'] as Map def childDataNode = new DataNodeBuilder().withDataspace('NCMP-Admin') - .withAnchor('AVC-Subscriptions').withXpath('/subscription-registry/subscription').build() + .withAnchor('AVC-Subscriptions').withXpath('/subscription-registry/subscription') + .withLeaves(leaves1).build() def engagedDataNode = new DataNodeBuilder().withDataspace('NCMP-Admin') .withAnchor('AVC-Subscriptions').withXpath('/subscription-registry') .withChildDataNodes([childDataNode]).build() - and: 'cps data service return existing data node' + and: 'cps data service return data node including a child data node' mockCpsDataService.getDataNodes(*_) >> [engagedDataNode] - when: 'the yangModelSubscriptionEvent is saved into db' + and: 'cps data service return data node for querying by xpaths' + mockCpsDataService.getDataNodesForMultipleXpaths(*_) >> [engagedDataNode] + when: 'the yang model subscription event is saved into db' objectUnderTest.saveSubscriptionEvent(yangModelSubscriptionEvent) - then: 'the cpsDataService update operation is called with the correct data' - 1 * mockCpsDataService.updateDataNodeAndDescendants(SUBSCRIPTION_DATASPACE_NAME, SUBSCRIPTION_ANCHOR_NAME, - SUBSCRIPTION_REGISTRY_PARENT, - '{"subscription":[{' + - '"topic":"some-topic",' + - '"predicates":{"datastore":"some-datastore","targetCmHandles":[{"cmHandleId":"cmhandle1","status":"PENDING"},{"cmHandleId":"cmhandle2","status":"PENDING"}]},' + - '"clientID":"some-client-id","subscriptionName":"some-subscription-name","isTagged":true}]}', + then: 'the cpsDataService save non-existing cm handle with the correct data' + 1 * mockCpsDataService.saveListElements(SUBSCRIPTION_DATASPACE_NAME, SUBSCRIPTION_ANCHOR_NAME, + SUBSCRIPTION_REGISTRY_PREDICATES_XPATH, '{"targetCmHandles":[{"cmHandleId":"cmhandle2","status":"PENDING"}]}', + NO_TIMESTAMP) + and: 'the cpsDataService update existing cm handle with the correct data' + 1 * mockCpsDataService.updateNodeLeaves(SUBSCRIPTION_DATASPACE_NAME, SUBSCRIPTION_ANCHOR_NAME, + SUBSCRIPTION_REGISTRY_PREDICATES_XPATH, '{"targetCmHandles":[{"cmHandleId":"cmhandle1","status":"PENDING"}]}', NO_TIMESTAMP) } diff --git a/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/utils/DataNodeHelperSpec.groovy b/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/utils/DataNodeHelperSpec.groovy index e527ae12bb..ee726a908e 100644 --- a/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/utils/DataNodeHelperSpec.groovy +++ b/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/utils/DataNodeHelperSpec.groovy @@ -20,6 +20,7 @@ package org.onap.cps.ncmp.api.impl.utils +import org.onap.cps.ncmp.api.impl.subscriptions.SubscriptionStatus import org.onap.cps.spi.model.DataNodeBuilder class DataNodeHelperSpec extends DataNodeBaseSpec { @@ -51,8 +52,22 @@ class DataNodeHelperSpec extends DataNodeBaseSpec { and: 'the nested data node is flatten and retrieves the leaves ' def leaves = DataNodeHelper.getDataNodeLeaves([dataNode]) when:'cm handle id to status is retrieved' - def result = DataNodeHelper.getCmHandleIdToStatus(leaves); + def result = DataNodeHelper.getCmHandleIdToStatus(leaves) then: 'the result list size is 3' result.size() == 3 + and: 'the result contains expected values' + result[0] as List == ['PENDING', 'CMHandle3'] + result[1] as List == ['ACCEPTED', 'CMHandle2'] + result[2] as List == ['REJECTED', 'CMHandle1'] + } + + def 'Get cm handle id to status map as expected from list of collection' () { + given: 'a list of collection' + def cmHandleCollection = [['PENDING', 'CMHandle3'], ['ACCEPTED', 'CMHandle2'], ['REJECTED', 'CMHandle1']] + when: 'the map is formed up with a method call' + def result = DataNodeHelper.getCmHandleIdToStatusMap(cmHandleCollection) + then: 'the map values are as expected' + result.keySet() == ['CMHandle3', 'CMHandle2', 'CMHandle1'] as Set + result.values() as List == [SubscriptionStatus.PENDING, SubscriptionStatus.ACCEPTED, SubscriptionStatus.REJECTED] } } diff --git a/cps-ncmp-service/src/test/resources/batchDataEvent.json b/cps-ncmp-service/src/test/resources/dataOperationEvent.json index 49eb273f58..42268c0ef3 100644 --- a/cps-ncmp-service/src/test/resources/batchDataEvent.json +++ b/cps-ncmp-service/src/test/resources/dataOperationEvent.json @@ -1,15 +1,15 @@ { - "event":{ - "batch-responses":[ + "data":{ + "responses":[ { "operationId":"1", "ids":[ "123", "124" ], - "status-code":1, - "status-message":"Batch operation success on the above cmhandle ids ", - "data":{ + "statusCode":1, + "statusMessage":"Batch operation success on the above cmhandle ids ", + "responseContent":{ "ietf-netconf-monitoring:netconf-state":{ "schemas":{ "schema":[ @@ -26,20 +26,6 @@ } } } - }, - { - "operationId":"101", - "ids":[ - "456", - "457" - ], - "status-code":101, - "status-message":"cmHandle(s) do not exist", - "data":{ - "error":{ - "message":"cmHandle(s) do not exist" - } - } } ] } |