Forward Subscription Information to DMI Plugin(s) 31/133431/9
authorlukegleeson <luke.gleeson@est.tech>
Thu, 23 Feb 2023 11:43:13 +0000 (11:43 +0000)
committerlukegleeson <luke.gleeson@est.tech>
Mon, 13 Mar 2023 16:23:22 +0000 (16:23 +0000)
Note: Implemented common EventsPublisher class

Issue-ID: CPS-1431
Signed-off-by: lukegleeson <luke.gleeson@est.tech>
Change-Id: I292a95f2c990a140f5fd63622ca4eba3f8284b9e

12 files changed:
cps-ncmp-events/src/main/resources/schemas/avc-subscription-event-v1.json
cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/event/EventsPublisher.java [moved from cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/event/lcm/LcmEventsPublisher.java with 73% similarity]
cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventConsumer.java
cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventForwarder.java [new file with mode: 0644]
cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/event/lcm/LcmEventsService.java
cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventConsumerSpec.groovy
cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventForwarderSpec.groovy [new file with mode: 0644]
cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/event/lcm/LcmEventsPublisherSpec.groovy
cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/event/lcm/LcmEventsServiceSpec.groovy
cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/kafka/MessagingBaseSpec.groovy
cps-ncmp-service/src/test/resources/avcSubscriptionCreationEvent.json
cps-service/src/main/java/org/onap/cps/spi/exceptions/OperationNotYetSupportedException.java [new file with mode: 0644]

index 5ab446c..feff48c 100644 (file)
@@ -29,6 +29,7 @@
     "event": {
       "description": "The event content.",
       "type": "object",
+      "javaType": "InnerSubscriptionEvent",
       "properties": {
         "subscription": {
           "description": "The subscription details.",
         ],
         "predicates": {
           "description": "Additional values to be added into the subscription",
-          "existingJavaType" : "java.util.Map<String,Object>",
-          "type" : "object"
-          }
+          "type" : "object",
+          "properties": {
+            "targets": {
+              "description": "CM Handles to be targeted by the subscription",
+              "type" : "array"
+            },
+            "datastore": {
+              "description": "datastore which is to be used by the subscription",
+              "type": "string"
+            },
+            "xpath-filter": {
+              "description": "filter to be applied to the CM Handles through this event",
+              "type": "string"
+            }
+          },
+          "required": ["datastore"]
         }
-      },
-      "required": [
+      }
+    },
+    "required": [
         "subscription",
         "dataType"
       ]
  * ============LICENSE_END=========================================================
  */
 
-package org.onap.cps.ncmp.api.impl.event.lcm;
+package org.onap.cps.ncmp.api.impl.event;
 
 import lombok.RequiredArgsConstructor;
 import lombok.extern.slf4j.Slf4j;
-import org.onap.ncmp.cmhandle.event.lcm.LcmEvent;
 import org.springframework.kafka.core.KafkaTemplate;
 import org.springframework.kafka.support.SendResult;
 import org.springframework.stereotype.Service;
@@ -30,36 +29,36 @@ import org.springframework.util.concurrent.ListenableFuture;
 import org.springframework.util.concurrent.ListenableFutureCallback;
 
 /**
- * LcmEventsPublisher to publish the LcmEvents on event of CREATE, UPDATE and DELETE.
+ * EventsPublisher to publish events.
  */
 
 @Slf4j
 @Service
 @RequiredArgsConstructor
-public class LcmEventsPublisher {
+public class EventsPublisher<T> {
 
-    private final KafkaTemplate<String, LcmEvent> lcmEventKafkaTemplate;
+    private final KafkaTemplate<String, T> eventKafkaTemplate;
 
     /**
      * LCM Event publisher.
      *
      * @param topicName valid topic name
      * @param eventKey  message key
-     * @param lcmEvent message payload
+     * @param event message payload
      */
-    public void publishEvent(final String topicName, final String eventKey, final LcmEvent lcmEvent) {
-        final ListenableFuture<SendResult<String, LcmEvent>> lcmEventFuture =
-                lcmEventKafkaTemplate.send(topicName, eventKey, lcmEvent);
+    public void publishEvent(final String topicName, final String eventKey, final T event) {
+        final ListenableFuture<SendResult<String, T>> eventFuture =
+                eventKafkaTemplate.send(topicName, eventKey, event);
 
-        lcmEventFuture.addCallback(new ListenableFutureCallback<>() {
+        eventFuture.addCallback(new ListenableFutureCallback<>() {
             @Override
             public void onFailure(final Throwable throwable) {
                 log.error("Unable to publish event to topic : {} due to {}", topicName, throwable.getMessage());
             }
 
             @Override
-            public void onSuccess(final SendResult<String, LcmEvent> sendResult) {
-                log.debug("Successfully published event to topic : {} , LcmEvent : {}",
+            public void onSuccess(final SendResult<String, T> sendResult) {
+                log.debug("Successfully published event to topic : {} , Event : {}",
                         sendResult.getRecordMetadata().topic(), sendResult.getProducerRecord().value());
             }
         });
index 92949cb..d08baac 100644 (file)
@@ -22,7 +22,10 @@ package org.onap.cps.ncmp.api.impl.event.avc;
 
 import lombok.RequiredArgsConstructor;
 import lombok.extern.slf4j.Slf4j;
+import org.onap.cps.ncmp.event.model.InnerSubscriptionEvent;
 import org.onap.cps.ncmp.event.model.SubscriptionEvent;
+import org.onap.cps.spi.exceptions.OperationNotYetSupportedException;
+import org.springframework.beans.factory.annotation.Value;
 import org.springframework.kafka.annotation.KafkaListener;
 import org.springframework.stereotype.Component;
 
@@ -32,6 +35,11 @@ import org.springframework.stereotype.Component;
 @RequiredArgsConstructor
 public class SubscriptionEventConsumer {
 
+    private final SubscriptionEventForwarder subscriptionEventForwarder;
+
+    @Value("${notification.enabled:true}")
+    private boolean notificationFeatureEnabled;
+
     /**
      * Consume the specified event.
      *
@@ -40,12 +48,21 @@ public class SubscriptionEventConsumer {
     @KafkaListener(topics = "${app.ncmp.avc.subscription-topic}",
             properties = {"spring.json.value.default.type=org.onap.cps.ncmp.event.model.SubscriptionEvent"})
     public void consumeSubscriptionEvent(final SubscriptionEvent subscriptionEvent) {
-        if ("CM".equals(subscriptionEvent.getEvent().getDataType().getDataCategory())) {
+        final InnerSubscriptionEvent event = subscriptionEvent.getEvent();
+        final String eventDatastore = event.getPredicates().getDatastore();
+        if (!(eventDatastore.equals("passthrough-running") || eventDatastore.equals("passthrough-operational"))) {
+            throw new OperationNotYetSupportedException(
+                "passthrough datastores are currently only supported for event subscriptions");
+        }
+        if ("CM".equals(event.getDataType().getDataCategory())) {
             log.debug("Consuming event {} ...", subscriptionEvent);
             if ("CREATE".equals(subscriptionEvent.getEventType().value())) {
-                log.info("Subscription for ClientID {} with name{} ...",
-                        subscriptionEvent.getEvent().getSubscription().getClientID(),
-                        subscriptionEvent.getEvent().getSubscription().getName());
+                log.info("Subscription for ClientID {} with name {} ...",
+                        event.getSubscription().getClientID(),
+                        event.getSubscription().getName());
+                if (notificationFeatureEnabled) {
+                    subscriptionEventForwarder.forwardCreateSubscriptionEvent(subscriptionEvent);
+                }
             }
         } else {
             log.trace("Non-CM subscription event ignored");
diff --git a/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventForwarder.java b/cps-ncmp-service/src/main/java/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventForwarder.java
new file mode 100644 (file)
index 0000000..635059b
--- /dev/null
@@ -0,0 +1,106 @@
+/*
+ * ============LICENSE_START=======================================================
+ *  Copyright (C) 2023 Nordix Foundation
+ *  ================================================================================
+ *  Licensed under the Apache License, Version 2.0 (the "License");
+ *  you may not use this file except in compliance with the License.
+ *  You may obtain a copy of the License at
+ *
+ *        http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+ *
+ *  SPDX-License-Identifier: Apache-2.0
+ *  ============LICENSE_END=========================================================
+ */
+
+package org.onap.cps.ncmp.api.impl.event.avc;
+
+import java.util.Collection;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.Objects;
+import java.util.stream.Collectors;
+import lombok.RequiredArgsConstructor;
+import lombok.extern.slf4j.Slf4j;
+import org.onap.cps.ncmp.api.impl.event.EventsPublisher;
+import org.onap.cps.ncmp.api.impl.operations.RequiredDmiService;
+import org.onap.cps.ncmp.api.impl.yangmodels.YangModelCmHandle;
+import org.onap.cps.ncmp.api.inventory.InventoryPersistence;
+import org.onap.cps.ncmp.event.model.SubscriptionEvent;
+import org.onap.cps.spi.exceptions.OperationNotYetSupportedException;
+import org.springframework.stereotype.Component;
+
+
+@Component
+@Slf4j
+@RequiredArgsConstructor
+public class SubscriptionEventForwarder {
+
+    private final InventoryPersistence inventoryPersistence;
+    private final EventsPublisher<SubscriptionEvent> eventsPublisher;
+
+    private static final String DMI_AVC_SUBSCRIPTION_TOPIC_PREFIX = "ncmp-dmi-cm-avc-subscription-";
+
+    /**
+     * Forward subscription event.
+     *
+     * @param subscriptionEvent the event to be forwarded
+     */
+    public void forwardCreateSubscriptionEvent(final SubscriptionEvent subscriptionEvent) {
+        final List<Object> cmHandleTargets = subscriptionEvent.getEvent().getPredicates().getTargets();
+        if (cmHandleTargets == null || cmHandleTargets.isEmpty()
+            || cmHandleTargets.stream().anyMatch(id -> ((String) id).contains("*"))) {
+            throw new OperationNotYetSupportedException(
+                "CMHandle targets are required. \"Wildcard\" operations are not yet supported");
+        }
+        final List<String> cmHandleTargetsAsStrings = cmHandleTargets.stream().map(
+            Objects::toString).collect(Collectors.toList());
+        final Collection<YangModelCmHandle> yangModelCmHandles =
+            inventoryPersistence.getYangModelCmHandles(cmHandleTargetsAsStrings);
+        final Map<String, Map<String, Map<String, String>>> dmiNameCmHandleMap =
+            organizeByDmiName(yangModelCmHandles);
+        dmiNameCmHandleMap.forEach((dmiName, cmHandlePropertiesMap) -> {
+            subscriptionEvent.getEvent().getPredicates().setTargets(Collections.singletonList(cmHandlePropertiesMap));
+            final String eventKey = createEventKey(subscriptionEvent, dmiName);
+            eventsPublisher.publishEvent(DMI_AVC_SUBSCRIPTION_TOPIC_PREFIX + dmiName, eventKey, subscriptionEvent);
+        });
+    }
+
+    private Map<String, Map<String, Map<String, String>>> organizeByDmiName(
+        final Collection<YangModelCmHandle> yangModelCmHandles) {
+        final Map<String, Map<String, Map<String, String>>> dmiNameCmHandlePropertiesMap = new HashMap<>();
+        yangModelCmHandles.forEach(cmHandle -> {
+            final String dmiName = cmHandle.resolveDmiServiceName(RequiredDmiService.DATA);
+            if (!dmiNameCmHandlePropertiesMap.containsKey(dmiName)) {
+                final Map<String, Map<String, String>> cmHandleDmiPropertiesMap = new HashMap<>();
+                cmHandleDmiPropertiesMap.put(cmHandle.getId(), dmiPropertiesAsMap(cmHandle));
+                dmiNameCmHandlePropertiesMap.put(cmHandle.getDmiDataServiceName(), cmHandleDmiPropertiesMap);
+            } else {
+                dmiNameCmHandlePropertiesMap.get(cmHandle.getDmiDataServiceName())
+                    .put(cmHandle.getId(), dmiPropertiesAsMap(cmHandle));
+            }
+        });
+        return dmiNameCmHandlePropertiesMap;
+    }
+
+    private String createEventKey(final SubscriptionEvent subscriptionEvent, final String dmiName) {
+        return subscriptionEvent.getEvent().getSubscription().getClientID()
+            + "-"
+            + subscriptionEvent.getEvent().getSubscription().getName()
+            + "-"
+            + dmiName;
+    }
+
+    public Map<String, String> dmiPropertiesAsMap(final YangModelCmHandle yangModelCmHandle) {
+        return yangModelCmHandle.getDmiProperties().stream().collect(
+            Collectors.toMap(YangModelCmHandle.Property::getName, YangModelCmHandle.Property::getValue));
+    }
+
+}
index a94d664..2eba830 100644 (file)
@@ -23,6 +23,7 @@ package org.onap.cps.ncmp.api.impl.event.lcm;
 import io.micrometer.core.annotation.Timed;
 import lombok.RequiredArgsConstructor;
 import lombok.extern.slf4j.Slf4j;
+import org.onap.cps.ncmp.api.impl.event.EventsPublisher;
 import org.onap.ncmp.cmhandle.event.lcm.LcmEvent;
 import org.springframework.beans.factory.annotation.Value;
 import org.springframework.kafka.KafkaException;
@@ -37,7 +38,7 @@ import org.springframework.stereotype.Service;
 @RequiredArgsConstructor
 public class LcmEventsService {
 
-    private final LcmEventsPublisher lcmEventsPublisher;
+    private final EventsPublisher<LcmEvent> eventsPublisher;
 
     @Value("${app.lcm.events.topic:ncmp-events}")
     private String topicName;
@@ -56,7 +57,7 @@ public class LcmEventsService {
     public void publishLcmEvent(final String cmHandleId, final LcmEvent lcmEvent) {
         if (notificationsEnabled) {
             try {
-                lcmEventsPublisher.publishEvent(topicName, cmHandleId, lcmEvent);
+                eventsPublisher.publishEvent(topicName, cmHandleId, lcmEvent);
             } catch (final KafkaException e) {
                 log.error("Unable to publish message to topic : {} and cause : {}", topicName, e.getMessage());
             }
index 20d60e3..7a9dade 100644 (file)
@@ -1,6 +1,6 @@
 /*
  * ============LICENSE_START=======================================================
- * Copyright (c) 2022 Nordix Foundation.
+ * Copyright (c) 2022-2023 Nordix Foundation.
  *  ================================================================================
  *  Licensed under the Apache License, Version 2.0 (the "License");
  *  you may not use this file except in compliance with the License.
@@ -24,29 +24,68 @@ import com.fasterxml.jackson.databind.ObjectMapper
 import org.onap.cps.ncmp.api.kafka.MessagingBaseSpec
 import org.onap.cps.ncmp.event.model.SubscriptionEvent
 import org.onap.cps.ncmp.utils.TestUtils
+import org.onap.cps.spi.exceptions.OperationNotYetSupportedException
 import org.onap.cps.utils.JsonObjectMapper
 import org.springframework.beans.factory.annotation.Autowired
 import org.springframework.boot.test.context.SpringBootTest
 
-@SpringBootTest(classes = [SubscriptionEventConsumer, ObjectMapper, JsonObjectMapper])
+@SpringBootTest(classes = [ObjectMapper, JsonObjectMapper])
 class SubscriptionEventConsumerSpec extends MessagingBaseSpec {
 
-    def objectUnderTest = new SubscriptionEventConsumer()
+    def subscriptionEventForwarder = Mock(SubscriptionEventForwarder)
+    def objectUnderTest = new SubscriptionEventConsumer(subscriptionEventForwarder)
 
     @Autowired
     JsonObjectMapper jsonObjectMapper
 
-    def 'Consume valid message'() {
+    def 'Consume and forward valid CM create message'() {
+        given: 'an event with data category CM'
+            def jsonData = TestUtils.getResourceFileContent('avcSubscriptionCreationEvent.json')
+            def testEventSent = jsonObjectMapper.convertJsonString(jsonData, SubscriptionEvent.class)
+        and: 'notifications are enabled'
+            objectUnderTest.notificationFeatureEnabled = true
+        when: 'the valid event is consumed'
+            objectUnderTest.consumeSubscriptionEvent(testEventSent)
+        then: 'the event is forwarded'
+            1 * subscriptionEventForwarder.forwardCreateSubscriptionEvent(testEventSent)
+    }
+
+    def 'Consume valid CM create message where notifications are disabled'() {
+        given: 'an event with data category CM'
+            def jsonData = TestUtils.getResourceFileContent('avcSubscriptionCreationEvent.json')
+            def testEventSent = jsonObjectMapper.convertJsonString(jsonData, SubscriptionEvent.class)
+        and: 'notifications are disabled'
+            objectUnderTest.notificationFeatureEnabled = false
+        when: 'the valid event is consumed'
+            objectUnderTest.consumeSubscriptionEvent(testEventSent)
+        then: 'the event is forwarded'
+            0 * subscriptionEventForwarder.forwardCreateSubscriptionEvent(testEventSent)
+    }
+
+    def 'Consume valid FM message'() {
         given: 'an event'
             def jsonData = TestUtils.getResourceFileContent('avcSubscriptionCreationEvent.json')
             def testEventSent = jsonObjectMapper.convertJsonString(jsonData, SubscriptionEvent.class)
-        and: 'dataCategory is set'
-            testEventSent.getEvent().getDataType().setDataCategory(dataCategory)
+        and: 'dataCategory is set to FM'
+            testEventSent.getEvent().getDataType().setDataCategory("FM")
         when: 'the valid event is consumed'
             objectUnderTest.consumeSubscriptionEvent(testEventSent)
         then: 'no exception is thrown'
             noExceptionThrown()
-        where: 'data category is changed'
-            dataCategory << [ 'CM' , 'FM' ]
+        and: 'No event is forwarded'
+            0 * subscriptionEventForwarder.forwardCreateSubscriptionEvent(*_)
+    }
+
+    def 'Consume event with wrong datastore causes an exception'() {
+        given: 'an event'
+            def jsonData = TestUtils.getResourceFileContent('avcSubscriptionCreationEvent.json')
+            def testEventSent = jsonObjectMapper.convertJsonString(jsonData, SubscriptionEvent.class)
+        and: 'datastore is set to a non passthrough datastore'
+            testEventSent.getEvent().getPredicates().setDatastore("operational")
+        when: 'the valid event is consumed'
+            objectUnderTest.consumeSubscriptionEvent(testEventSent)
+        then: 'an operation not yet supported exception is thrown'
+            thrown(OperationNotYetSupportedException)
     }
+
 }
diff --git a/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventForwarderSpec.groovy b/cps-ncmp-service/src/test/groovy/org/onap/cps/ncmp/api/impl/event/avc/SubscriptionEventForwarderSpec.groovy
new file mode 100644 (file)
index 0000000..f9e801d
--- /dev/null
@@ -0,0 +1,96 @@
+/*
+ * ============LICENSE_START=======================================================
+ * Copyright (c) 2023 Nordix Foundation.
+ *  ================================================================================
+ *  Licensed under the Apache License, Version 2.0 (the "License");
+ *  you may not use this file except in compliance with the License.
+ *  You may obtain a copy of the License at
+ *
+ *        http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an 'AS IS' BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+ *
+ *  SPDX-License-Identifier: Apache-2.0
+ *  ============LICENSE_END=========================================================
+ */
+
+package org.onap.cps.ncmp.api.impl.event.avc
+
+import com.fasterxml.jackson.databind.ObjectMapper
+import org.onap.cps.ncmp.api.impl.event.EventsPublisher
+import org.onap.cps.ncmp.api.impl.yangmodels.YangModelCmHandle
+import org.onap.cps.ncmp.api.inventory.InventoryPersistence
+import org.onap.cps.ncmp.api.kafka.MessagingBaseSpec
+import org.onap.cps.ncmp.event.model.SubscriptionEvent
+import org.onap.cps.ncmp.utils.TestUtils
+import org.onap.cps.spi.exceptions.OperationNotYetSupportedException
+import org.onap.cps.utils.JsonObjectMapper
+import org.springframework.beans.factory.annotation.Autowired
+import org.springframework.boot.test.context.SpringBootTest
+import org.springframework.kafka.core.KafkaTemplate
+import org.springframework.util.concurrent.ListenableFuture;
+
+@SpringBootTest(classes = [ObjectMapper, JsonObjectMapper])
+class SubscriptionEventForwarderSpec extends MessagingBaseSpec {
+
+    def mockInventoryPersistence = Mock(InventoryPersistence)
+    def mockSubscriptionEventPublisher = Mock(EventsPublisher<SubscriptionEvent>)
+    def objectUnderTest = new SubscriptionEventForwarder(mockInventoryPersistence, mockSubscriptionEventPublisher)
+
+    @Autowired
+    JsonObjectMapper jsonObjectMapper
+
+    def 'Forward valid CM create subscription'() {
+        given: 'an event'
+            def jsonData = TestUtils.getResourceFileContent('avcSubscriptionCreationEvent.json')
+            def testEventSent = jsonObjectMapper.convertJsonString(jsonData, SubscriptionEvent.class)
+        and: 'the InventoryPersistence returns private properties for the supplied CM Handles'
+            1 * mockInventoryPersistence.getYangModelCmHandles(["CMHandle1", "CMHandle2", "CMHandle3"]) >> [
+                createYangModelCmHandleWithDmiProperty(1, 1,"shape","circle"),
+                createYangModelCmHandleWithDmiProperty(2, 1,"shape","square"),
+                createYangModelCmHandleWithDmiProperty(3, 2,"shape","triangle")
+            ]
+        when: 'the valid event is forwarded'
+            objectUnderTest.forwardCreateSubscriptionEvent(testEventSent)
+        then: 'the event is forwarded twice with the CMHandle private properties and provides a valid listenable future'
+            1 * mockSubscriptionEventPublisher.publishEvent("ncmp-dmi-cm-avc-subscription-DMIName1", "SCO-9989752-cm-subscription-001-DMIName1",
+                subscriptionEvent -> {
+                    Map targets = subscriptionEvent.getEvent().getPredicates().getTargets().get(0)
+                    targets["CMHandle1"] == ["shape":"circle"]
+                    targets["CMHandle2"] == ["shape":"square"]
+                }
+            )
+            1 * mockSubscriptionEventPublisher.publishEvent("ncmp-dmi-cm-avc-subscription-DMIName2", "SCO-9989752-cm-subscription-001-DMIName2",
+                subscriptionEvent -> {
+                    Map targets = subscriptionEvent.getEvent().getPredicates().getTargets().get(0)
+                    targets["CMHandle3"] == ["shape":"triangle"]
+                }
+            )
+    }
+
+    def 'Forward CM create subscription where target CM Handles are #scenario'() {
+        given: 'an event'
+            def jsonData = TestUtils.getResourceFileContent('avcSubscriptionCreationEvent.json')
+            def testEventSent = jsonObjectMapper.convertJsonString(jsonData, SubscriptionEvent.class)
+        and: 'the target CMHandles are set to #scenario'
+            testEventSent.getEvent().getPredicates().setTargets(invalidTargets)
+        when: 'the event is forwarded'
+            objectUnderTest.forwardCreateSubscriptionEvent(testEventSent)
+        then: 'an operation not yet supported exception is thrown'
+            thrown(OperationNotYetSupportedException)
+        where:
+            scenario   | invalidTargets
+            'null'     | null
+            'empty'    | []
+            'wildcard' | ['CMHandle*']
+    }
+
+    static def createYangModelCmHandleWithDmiProperty(id, dmiId,propertyName, propertyValue) {
+        return new YangModelCmHandle(id:"CMHandle" + id, dmiDataServiceName: "DMIName" + dmiId, dmiProperties: [new YangModelCmHandle.Property(propertyName,propertyValue)])
+    }
+
+}
index 61bf33d..f5b58a7 100644 (file)
@@ -1,6 +1,6 @@
 /*
  * ============LICENSE_START=======================================================
- * Copyright (C) 2022 Nordix Foundation
+ * Copyright (C) 2022-2023 Nordix Foundation
  * ================================================================================
  * Licensed under the Apache License, Version 2.0 (the "License");
  * you may not use this file except in compliance with the License.
@@ -22,6 +22,7 @@ package org.onap.cps.ncmp.api.impl.event.lcm
 
 import com.fasterxml.jackson.databind.ObjectMapper
 import org.apache.kafka.clients.consumer.KafkaConsumer
+import org.onap.cps.ncmp.api.impl.event.EventsPublisher
 import org.onap.cps.ncmp.api.kafka.MessagingBaseSpec
 import org.onap.cps.ncmp.utils.TestUtils
 import org.onap.cps.utils.JsonObjectMapper
@@ -35,7 +36,7 @@ import org.testcontainers.spock.Testcontainers
 
 import java.time.Duration
 
-@SpringBootTest(classes = [LcmEventsPublisher, ObjectMapper, JsonObjectMapper])
+@SpringBootTest(classes = [EventsPublisher, ObjectMapper, JsonObjectMapper])
 @Testcontainers
 @DirtiesContext
 class LcmEventsPublisherSpec extends MessagingBaseSpec {
@@ -45,7 +46,7 @@ class LcmEventsPublisherSpec extends MessagingBaseSpec {
     def testTopic = 'ncmp-events-test'
 
     @SpringBean
-    LcmEventsPublisher lcmEventsPublisher = new LcmEventsPublisher(kafkaTemplate)
+    EventsPublisher<LcmEvent> lcmEventsPublisher = new EventsPublisher(kafkaTemplate)
 
     @Autowired
     JsonObjectMapper jsonObjectMapper
index ef399e1..4c632dd 100644 (file)
@@ -1,6 +1,6 @@
 /*
  * ============LICENSE_START=======================================================
- * Copyright (C) 2022 Nordix Foundation
+ * Copyright (C) 2022-2023 Nordix Foundation
  * ================================================================================
  * Licensed under the Apache License, Version 2.0 (the "License");
  * you may not use this file except in compliance with the License.
 
 package org.onap.cps.ncmp.api.impl.event.lcm
 
+import org.onap.cps.ncmp.api.impl.event.EventsPublisher
 import org.onap.ncmp.cmhandle.event.lcm.LcmEvent
 import org.springframework.kafka.KafkaException
 import spock.lang.Specification
 
 class LcmEventsServiceSpec extends Specification {
 
-    def mockLcmEventsPublisher = Mock(LcmEventsPublisher)
+    def mockLcmEventsPublisher = Mock(EventsPublisher)
 
     def objectUnderTest = new LcmEventsService(mockLcmEventsPublisher)
 
index bb0ce87..337178e 100644 (file)
@@ -22,6 +22,7 @@ package org.onap.cps.ncmp.api.kafka
 
 import org.apache.kafka.common.serialization.StringDeserializer
 import org.apache.kafka.common.serialization.StringSerializer
+import org.spockframework.spring.SpringBean
 import org.springframework.kafka.core.DefaultKafkaProducerFactory
 import org.springframework.kafka.core.KafkaTemplate
 import org.springframework.kafka.support.serializer.JsonSerializer
@@ -62,7 +63,8 @@ class MessagingBaseSpec extends Specification {
         ]
     }
 
-    def kafkaTemplate = new KafkaTemplate<>(new DefaultKafkaProducerFactory<Integer, String>(producerConfigProperties()))
+    @SpringBean
+    KafkaTemplate kafkaTemplate = new KafkaTemplate<>(new DefaultKafkaProducerFactory<Integer, String>(producerConfigProperties()))
 
     @DynamicPropertySource
     static void registerKafkaProperties(DynamicPropertyRegistry dynamicPropertyRegistry) {
index 1d84c3a..63fca1f 100644 (file)
       "schemaVersion": "1.0"
     },
     "predicates": {
-      "datastore": "passthrough-operational",
-      "datastore-xpath-filter": "//_3gpp-nr-nrm-gnbdufunction:GNBDUFunction/ ",
-      "_3gpp-nr-nrm-nrcelldu": "NRCellDU"
-
-    }
+      "targets" :  ["CMHandle1", "CMHandle2", "CMHandle3"],
+      "datastore": "passthrough-running",
+      "xpath-filter": "//_3gpp-nr-nrm-gnbdufunction:GNBDUFunction/_3gpp-nr-nrm-nrcelldu:NRCellDU/ | //_3gpp-nr-nrm-gnbcuupfunction:GNBCUUPFunction// | //_3gpp-nr-nrm-gnbcucpfunction:GNBCUCPFunction/_3gpp-nr-nrm-nrcelldu:NRCellCU// | //_3gpp-nr-nrm-nrsectorcarrier:NRSectorCarrier//"
   }
+
+}
 }
\ No newline at end of file
diff --git a/cps-service/src/main/java/org/onap/cps/spi/exceptions/OperationNotYetSupportedException.java b/cps-service/src/main/java/org/onap/cps/spi/exceptions/OperationNotYetSupportedException.java
new file mode 100644 (file)
index 0000000..6a4e2a0
--- /dev/null
@@ -0,0 +1,40 @@
+/*
+ *  ============LICENSE_START=======================================================
+ *  Copyright (C) 2023 Nordix Foundation
+ *  ================================================================================
+ *  Licensed under the Apache License, Version 2.0 (the "License");
+ *  you may not use this file except in compliance with the License.
+ *  You may obtain a copy of the License at
+ *
+ *        http://www.apache.org/licenses/LICENSE-2.0
+ *
+ *  Unless required by applicable law or agreed to in writing, software
+ *  distributed under the License is distributed on an "AS IS" BASIS,
+ *  WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+ *  See the License for the specific language governing permissions and
+ *  limitations under the License.
+ *
+ *  SPDX-License-Identifier: Apache-2.0
+ *  ============LICENSE_END=========================================================
+ */
+
+package org.onap.cps.spi.exceptions;
+
+/**
+ * Operation Not Yet Supported Exception.
+ * Indicates the operation is not supported and has intention to be supported in the future.
+ */
+
+public class OperationNotYetSupportedException extends CpsException {
+
+    private static final long serialVersionUID = 1517903069236383746L;
+
+    /**
+     * Constructor.
+     *
+     * @param details reason for the exception
+     */
+    public OperationNotYetSupportedException(final String details) {
+        super("Operation Not Yet Supported Exception", details);
+    }
+}