aboutsummaryrefslogtreecommitdiffstats
path: root/ms
diff options
context:
space:
mode:
authorJozsef Csongvai <jozsef.csongvai@bell.ca>2020-06-15 08:42:08 -0400
committerKAPIL SINGAL <ks220y@att.com>2020-06-19 13:18:51 +0000
commit3579d15b41d9f786650f76c3b6a98d28f0052f1a (patch)
tree56d00d692f3569c8a705261a9184b06feefb7afe /ms
parent18174bb951a8193e5c310969fb85684b3145fd0e (diff)
Implement nodetemplate locking feature
Enables locking execution of a nodetemplate using a lock key and lock acquire timeout. Issue-ID: CCSDK-2460 Change-Id: I308d4d89dab44b7f7a766d5b62258e67b051eab1 Signed-off-by: Jozsef Csongvai <jozsef.csongvai@bell.ca>
Diffstat (limited to 'ms')
-rw-r--r--ms/blueprintsprocessor/modules/blueprints/blueprint-core/src/main/kotlin/org/onap/ccsdk/cds/controllerblueprints/core/data/BluePrintModel.kt7
-rw-r--r--ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensions.kt29
-rw-r--r--ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/HazlecastClusterService.kt10
-rw-r--r--ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/service/BluePrintClusterService.kt2
-rw-r--r--ms/blueprintsprocessor/modules/commons/processor-core/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensionsTest.kt74
-rw-r--r--ms/blueprintsprocessor/modules/services/execution-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/AbstractComponentFunction.kt33
-rw-r--r--ms/blueprintsprocessor/modules/services/execution-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/scripts/AbstractComponentFunctionTest.kt135
-rw-r--r--ms/blueprintsprocessor/modules/services/workflow-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionService.kt6
-rw-r--r--ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BluePrintWorkflowExecutionServiceImplTest.kt5
-rw-r--r--ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BlueprintServiceLogicTest.kt5
-rw-r--r--ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/DGWorkflowExecutionServiceTest.kt5
-rw-r--r--ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/ImperativeWorkflowExecutionServiceTest.kt3
-rw-r--r--ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionServiceTest.kt7
13 files changed, 287 insertions, 34 deletions
diff --git a/ms/blueprintsprocessor/modules/blueprints/blueprint-core/src/main/kotlin/org/onap/ccsdk/cds/controllerblueprints/core/data/BluePrintModel.kt b/ms/blueprintsprocessor/modules/blueprints/blueprint-core/src/main/kotlin/org/onap/ccsdk/cds/controllerblueprints/core/data/BluePrintModel.kt
index e0333997d..24901257e 100644
--- a/ms/blueprintsprocessor/modules/blueprints/blueprint-core/src/main/kotlin/org/onap/ccsdk/cds/controllerblueprints/core/data/BluePrintModel.kt
+++ b/ms/blueprintsprocessor/modules/blueprints/blueprint-core/src/main/kotlin/org/onap/ccsdk/cds/controllerblueprints/core/data/BluePrintModel.kt
@@ -24,6 +24,7 @@ import com.fasterxml.jackson.annotation.JsonPropertyOrder
import com.fasterxml.jackson.databind.JsonNode
import io.swagger.annotations.ApiModelProperty
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
+import org.onap.ccsdk.cds.controllerblueprints.core.asJsonType
/**
*
@@ -223,6 +224,12 @@ class Implementation {
var operationHost: String = BluePrintConstants.PROPERTY_SELF
// Timeout value in seconds
var timeout: Int = 180
+ var lock: LockAssignment? = null
+}
+
+class LockAssignment {
+ lateinit var key: JsonNode
+ var acquireTimeout: JsonNode = Integer(180).asJsonType()
}
/*
diff --git a/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensions.kt b/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensions.kt
index 85d9d5c27..81fc0d709 100644
--- a/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensions.kt
+++ b/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensions.kt
@@ -17,9 +17,15 @@
package org.onap.ccsdk.cds.blueprintsprocessor.core.cluster
import com.hazelcast.cluster.Member
+import kotlinx.coroutines.GlobalScope
+import kotlinx.coroutines.newSingleThreadContext
+import kotlinx.coroutines.withContext
import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.ClusterLock
import org.onap.ccsdk.cds.blueprintsprocessor.core.service.ClusterMember
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
+import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintException
+import org.onap.ccsdk.cds.controllerblueprints.core.MDCContext
import org.onap.ccsdk.cds.controllerblueprints.core.service.BluePrintDependencyService
/**
@@ -44,3 +50,26 @@ fun Member.toClusterMember(): ClusterMember {
memberAddress = this.address.toString()
)
}
+
+/**
+ * This function will try to acquire the lock and then execute the provided block.
+ * If the lock cannot be acquired within timeout, a BlueprintException will be thrown.
+ *
+ * Since a lock can only be unlocked by the the thread which acquired the lock,
+ * this function will confine coroutines within the block to a dedicated thread.
+ */
+suspend fun <R> ClusterLock.executeWithLock(acquireLockTimeout: Long, block: suspend () -> R): R {
+ val lock = this
+ return newSingleThreadContext(lock.name()).use {
+ withContext(GlobalScope.coroutineContext[MDCContext]?.plus(it) ?: it) {
+ if (lock.tryLock(acquireLockTimeout)) {
+ try {
+ block()
+ } finally {
+ lock.unLock()
+ }
+ } else
+ throw BluePrintException("Failed to acquire lock within timeout")
+ }
+ }
+}
diff --git a/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/HazlecastClusterService.kt b/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/HazlecastClusterService.kt
index a58c077fa..feb2a8e2a 100644
--- a/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/HazlecastClusterService.kt
+++ b/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/HazlecastClusterService.kt
@@ -133,7 +133,7 @@ open class HazlecastClusterService : BluePrintClusterService {
}
override fun clusterJoined(): Boolean {
- return hazelcast.lifecycleService.isRunning
+ return ::hazelcast.isInitialized && hazelcast.lifecycleService.isRunning
}
override suspend fun masterMember(partitionGroup: String): ClusterMember {
@@ -225,21 +225,21 @@ open class BlueprintsClusterMembershipListener() :
open class ClusterLockImpl(private val hazelcast: HazelcastInstance, private val name: String) : ClusterLock {
private val log = logger(ClusterLockImpl::class)
- lateinit var distributedLock: FencedLock
+ private val distributedLock: FencedLock = hazelcast.cpSubsystem.getLock(name)
override fun name(): String {
return distributedLock.name
}
override suspend fun lock() {
- distributedLock = hazelcast.cpSubsystem.getLock(name)
distributedLock.lock()
log.trace("Cluster lock($name) created..")
}
override suspend fun tryLock(timeout: Long): Boolean {
- distributedLock = hazelcast.cpSubsystem.getLock(name)
return distributedLock.tryLock(timeout, TimeUnit.MILLISECONDS)
+ .also { if (it) log.trace("Cluster lock acquired: $name")
+ else log.trace("Failed to acquire Cluster lock $name within timeout $timeout") }
}
override suspend fun unLock() {
@@ -255,14 +255,12 @@ open class ClusterLockImpl(private val hazelcast: HazelcastInstance, private val
}
override suspend fun fenceLock(): String {
- distributedLock = hazelcast.cpSubsystem.getLock(name)
val fence = distributedLock.lockAndGetFence()
log.trace("Cluster lock($name) fence($fence) created..")
return fence.toString()
}
override suspend fun tryFenceLock(timeout: Long): String {
- distributedLock = hazelcast.cpSubsystem.getLock(name)
return distributedLock.tryLockAndGetFence(timeout, TimeUnit.MILLISECONDS).toString()
}
diff --git a/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/service/BluePrintClusterService.kt b/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/service/BluePrintClusterService.kt
index 53f18d38a..9725553a5 100644
--- a/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/service/BluePrintClusterService.kt
+++ b/ms/blueprintsprocessor/modules/commons/processor-core/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/service/BluePrintClusterService.kt
@@ -80,3 +80,5 @@ interface ClusterLock {
fun isLocked(): Boolean
fun close()
}
+
+const val CDS_LOCK_GROUP = "cds-lock"
diff --git a/ms/blueprintsprocessor/modules/commons/processor-core/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensionsTest.kt b/ms/blueprintsprocessor/modules/commons/processor-core/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensionsTest.kt
new file mode 100644
index 000000000..7ef4eb49b
--- /dev/null
+++ b/ms/blueprintsprocessor/modules/commons/processor-core/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/core/cluster/BluePrintClusterExtensionsTest.kt
@@ -0,0 +1,74 @@
+/*
+ * Copyright © 2019 Bell Canada.
+ *
+ * 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.
+ */
+
+package org.onap.ccsdk.cds.blueprintsprocessor.core.cluster
+
+import io.mockk.every
+import io.mockk.mockk
+import io.mockk.verify
+import kotlinx.coroutines.runBlocking
+import org.junit.Before
+import org.junit.Test
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.ClusterLock
+import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintException
+import java.lang.RuntimeException
+import kotlin.test.assertEquals
+
+class BluePrintClusterExtensionsTest {
+
+ private lateinit var clusterLockMock: ClusterLock
+
+ @Before
+ fun setup() {
+ clusterLockMock = mockk()
+ every { clusterLockMock.name() } returns "mock-lock"
+ }
+
+ @Test
+ fun `executeWithLock - should call unlock and return block result`() {
+ runBlocking {
+ every { runBlocking { clusterLockMock.tryLock(more(0L)) } } returns true
+ every { runBlocking { clusterLockMock.unLock() } } returns Unit
+
+ val result = clusterLockMock.executeWithLock(1_000) { "result" }
+
+ verify { runBlocking { clusterLockMock.unLock() } }
+ assertEquals("result", result)
+ }
+ }
+
+ @Test
+ fun `executeWithLock - should call unlock even when block throws exception`() {
+ runBlocking {
+ every { runBlocking { clusterLockMock.tryLock(more(0L)) } } returns true
+ every { runBlocking { clusterLockMock.unLock() } } returns Unit
+
+ try {
+ clusterLockMock.executeWithLock(1_000) { throw RuntimeException("It crashed") }
+ } catch (e: Exception) { }
+
+ verify { runBlocking { clusterLockMock.unLock() } }
+ }
+ }
+
+ @Test(expected = BluePrintException::class)
+ fun `executeWithLock - should throw exception when lock was not acquired within timeout`() {
+ runBlocking {
+ every { runBlocking { clusterLockMock.tryLock(eq(0L)) } } returns false
+ clusterLockMock.executeWithLock(0) { "Will not run" }
+ }
+ }
+}
diff --git a/ms/blueprintsprocessor/modules/services/execution-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/AbstractComponentFunction.kt b/ms/blueprintsprocessor/modules/services/execution-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/AbstractComponentFunction.kt
index aa39a1dd0..211bf76fb 100644
--- a/ms/blueprintsprocessor/modules/services/execution-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/AbstractComponentFunction.kt
+++ b/ms/blueprintsprocessor/modules/services/execution-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/AbstractComponentFunction.kt
@@ -23,9 +23,14 @@ import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceInpu
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceOutput
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.Status
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.StepData
+import org.onap.ccsdk.cds.blueprintsprocessor.core.cluster.executeWithLock
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.CDS_LOCK_GROUP
import org.onap.ccsdk.cds.controllerblueprints.common.api.EventType
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintProcessorException
+import org.onap.ccsdk.cds.controllerblueprints.core.asJsonType
+import org.onap.ccsdk.cds.controllerblueprints.core.checkNotBlank
import org.onap.ccsdk.cds.controllerblueprints.core.checkNotEmpty
import org.onap.ccsdk.cds.controllerblueprints.core.data.Implementation
import org.onap.ccsdk.cds.controllerblueprints.core.getAsString
@@ -49,6 +54,7 @@ abstract class AbstractComponentFunction : BlueprintFunctionNode<ExecutionServic
lateinit var executionServiceInput: ExecutionServiceInput
var executionServiceOutput = ExecutionServiceOutput()
lateinit var bluePrintRuntimeService: BluePrintRuntimeService<*>
+ lateinit var bluePrintClusterService: BluePrintClusterService
lateinit var implementation: Implementation
lateinit var processId: String
lateinit var workflowName: String
@@ -95,6 +101,22 @@ abstract class AbstractComponentFunction : BlueprintFunctionNode<ExecutionServic
implementation = bluePrintRuntimeService.bluePrintContext()
.nodeTemplateOperationImplementation(nodeTemplateName, interfaceName, operationName)
?: Implementation()
+
+ /** Resolve and validate lock properties */
+ implementation.lock?.apply {
+ val resolvedValues = bluePrintRuntimeService.resolvePropertyAssignments(
+ nodeTemplateName,
+ interfaceName,
+ mutableMapOf("key" to this.key, "acquireTimeout" to this.acquireTimeout))
+ this.key = resolvedValues["key"] ?: "".asJsonType()
+ this.acquireTimeout = resolvedValues["acquireTimeout"] ?: "".asJsonType()
+
+ checkNotBlank(this.key.textValue()) { "Failed to resolve lock key" }
+ check(this.acquireTimeout.isInt && this.acquireTimeout.intValue() >= 0) {
+ "Failed to resolve lock acquireTimeout - must be a positive integer"
+ }
+ }
+
check(this::implementation.isInitialized) { "failed to prepare implementation" }
val operationResolvedProperties = bluePrintRuntimeService
@@ -131,8 +153,17 @@ abstract class AbstractComponentFunction : BlueprintFunctionNode<ExecutionServic
}
override suspend fun applyNB(executionServiceInput: ExecutionServiceInput): ExecutionServiceOutput {
+ prepareRequestNB(executionServiceInput)
+ return implementation.lock?.let {
+ bluePrintClusterService.clusterLock("${it.key.textValue()}@$CDS_LOCK_GROUP")
+ .executeWithLock(it.acquireTimeout.intValue().times(1000).toLong()) {
+ applyNBWithTimeout(executionServiceInput)
+ }
+ } ?: applyNBWithTimeout(executionServiceInput)
+ }
+
+ private suspend fun applyNBWithTimeout(executionServiceInput: ExecutionServiceInput): ExecutionServiceOutput {
try {
- prepareRequestNB(executionServiceInput)
withTimeout((implementation.timeout * 1000).toLong()) {
log.debug("DEBUG::: AbstractComponentFunction.withTimeout section ${implementation.timeout} seconds")
processNB(executionServiceInput)
diff --git a/ms/blueprintsprocessor/modules/services/execution-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/scripts/AbstractComponentFunctionTest.kt b/ms/blueprintsprocessor/modules/services/execution-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/scripts/AbstractComponentFunctionTest.kt
index 3caa061e8..e0b690573 100644
--- a/ms/blueprintsprocessor/modules/services/execution-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/scripts/AbstractComponentFunctionTest.kt
+++ b/ms/blueprintsprocessor/modules/services/execution-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/execution/scripts/AbstractComponentFunctionTest.kt
@@ -25,6 +25,7 @@ import com.fasterxml.jackson.databind.ObjectMapper
import com.fasterxml.jackson.databind.node.ObjectNode
import io.mockk.every
import io.mockk.mockk
+import io.mockk.verify
import kotlinx.coroutines.runBlocking
import org.junit.Test
import org.junit.runner.RunWith
@@ -32,13 +33,18 @@ import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ActionIdentifiers
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.CommonHeader
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceInput
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.StepData
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.CDS_LOCK_GROUP
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.ClusterLock
import org.onap.ccsdk.cds.blueprintsprocessor.services.execution.AbstractComponentFunction
import org.onap.ccsdk.cds.blueprintsprocessor.services.execution.ComponentFunctionScriptingService
import org.onap.ccsdk.cds.blueprintsprocessor.services.execution.nodeTypeComponentScriptExecutor
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintTypes
import org.onap.ccsdk.cds.controllerblueprints.core.asJsonPrimitive
+import org.onap.ccsdk.cds.controllerblueprints.core.asJsonType
import org.onap.ccsdk.cds.controllerblueprints.core.data.Implementation
+import org.onap.ccsdk.cds.controllerblueprints.core.data.LockAssignment
import org.onap.ccsdk.cds.controllerblueprints.core.normalizedPathName
import org.onap.ccsdk.cds.controllerblueprints.core.scripts.BluePrintScriptsServiceImpl
import org.onap.ccsdk.cds.controllerblueprints.core.service.BluePrintContext
@@ -61,14 +67,20 @@ import kotlin.test.assertNotNull
)
class AbstractComponentFunctionTest {
+ lateinit var bluePrintRuntimeService: DefaultBluePrintRuntimeService
lateinit var blueprintContext: BluePrintContext
+ lateinit var blueprintClusterService: BluePrintClusterService
@Autowired
lateinit var compSvc: ComponentFunctionScriptingService
@BeforeTest
fun init() {
- blueprintContext = mockk<BluePrintContext>()
+ bluePrintRuntimeService = mockk()
+ blueprintContext = mockk()
+ blueprintClusterService = mockk()
+ every { bluePrintRuntimeService.bluePrintContext() } returns blueprintContext
+
every { blueprintContext.rootPath } returns normalizedPathName("target")
every {
blueprintContext.nodeTemplateOperationImplementation(
@@ -80,15 +92,14 @@ class AbstractComponentFunctionTest {
@Test
fun testAbstractComponent() {
runBlocking {
- val bluePrintRuntime = mockk<DefaultBluePrintRuntimeService>("1234")
val samp = SampleComponent()
val comp = samp as AbstractComponentFunction
- comp.bluePrintRuntimeService = bluePrintRuntime
+ comp.bluePrintRuntimeService = bluePrintRuntimeService
comp.stepName = "sample-step"
assertNotNull(comp, "failed to get kotlin instance")
- val input = getMockedInput(bluePrintRuntime)
+ val input = getMockedInput(bluePrintRuntimeService)
val output = comp.applyNB(input)
@@ -115,16 +126,14 @@ class AbstractComponentFunctionTest {
@Test
fun testAbstractScriptComponent() {
runBlocking {
- val bluePrintRuntime = mockk<DefaultBluePrintRuntimeService>("1234")
val samp = SampleRestconfComponent(compSvc)
val comp = samp as AbstractComponentFunction
- comp.bluePrintRuntimeService = bluePrintRuntime
+ comp.bluePrintRuntimeService = bluePrintRuntimeService
comp.stepName = "sample-step"
assertNotNull(comp, "failed to get kotlin instance")
- val input = getMockedInput(bluePrintRuntime)
- val inp = getMockedContext()
+ val input = getMockedInput(bluePrintRuntimeService)
val output = comp.applyNB(input)
@@ -135,17 +144,103 @@ class AbstractComponentFunctionTest {
}
}
- /**
- * Mocked input for abstract function test.
- */
- private fun getMockedContext() {
- val operationOutputs = hashMapOf<String, JsonNode>()
+ @Test
+ fun testComponentScriptExecutorNodeType() {
+ val componentScriptExecutor = BluePrintTypes.nodeTypeComponentScriptExecutor()
+ assertNotNull(componentScriptExecutor.interfaces, "failed to get interface operations")
+ }
+
+ @Test
+ fun `prepareRequestNB should resolve lock properties`() {
+ val implementation = Implementation().apply {
+ this.lock = LockAssignment().apply {
+ this.key = """ {"get_input": "lock-key"} """.asJsonPrimitive()
+ }
+ }
every {
- blueprintContext.name()
- } returns "SampleTest"
+ blueprintContext.nodeTemplateOperationImplementation(any(), any(), any())
+ } returns implementation
+
every {
- blueprintContext.version()
- } returns "SampleScriptComponent"
+ bluePrintRuntimeService.resolvePropertyAssignments(any(), any(), any())
+ } returns mutableMapOf(
+ "key" to "abc-123-def-456".asJsonType(),
+ "acquireTimeout" to implementation.lock!!.acquireTimeout
+ )
+
+ val component: AbstractComponentFunction = SampleComponent()
+ component.bluePrintRuntimeService = bluePrintRuntimeService
+ component.bluePrintClusterService = blueprintClusterService
+
+ runBlocking {
+ component.prepareRequestNB(getMockedInput(bluePrintRuntimeService))
+ }
+
+ val resolvedLock = component.implementation.lock!!
+
+ assertEquals("abc-123-def-456", resolvedLock.key.textValue())
+ // default value
+ assertEquals(180, resolvedLock.acquireTimeout.intValue())
+ }
+
+ @Test(expected = Exception::class)
+ fun `prepareRequestNB should throw exception if it fails to resolve lock key`() {
+ every {
+ blueprintContext.nodeTemplateOperationImplementation(any(), any(), any())
+ } returns Implementation().apply { this.lock = LockAssignment() }
+
+ every {
+ bluePrintRuntimeService.resolvePropertyAssignments(any(), any(), any())
+ } returns mutableMapOf("key" to "".asJsonType(),
+ "acquireTimeout" to Integer(360).asJsonType())
+
+ val component: AbstractComponentFunction = SampleComponent()
+ component.bluePrintRuntimeService = bluePrintRuntimeService
+ component.bluePrintClusterService = blueprintClusterService
+
+ runBlocking {
+ component.prepareRequestNB(getMockedInput(bluePrintRuntimeService))
+ }
+ }
+
+ @Test
+ fun `applyNB - when lock is present use ClusterLock`() {
+
+ val lockName = "testing-lock"
+
+ every {
+ blueprintContext.nodeTemplateOperationImplementation(any(), any(), any())
+ } returns Implementation().apply {
+ this.lock = LockAssignment().apply { this.key = lockName.asJsonType() }
+ }
+
+ every {
+ bluePrintRuntimeService.resolvePropertyAssignments(any(), any(), any())
+ } returns mutableMapOf("key" to lockName.asJsonType(),
+ "acquireTimeout" to Integer(180).asJsonType())
+
+ val clusterLock: ClusterLock = mockk()
+
+ every { clusterLock.name() } returns lockName
+ every { runBlocking { clusterLock.tryLock(any()) } } returns true
+ every { runBlocking { clusterLock.unLock() } } returns Unit
+
+ every {
+ runBlocking { blueprintClusterService.clusterLock(any()) }
+ } returns clusterLock
+
+ val component: AbstractComponentFunction = SampleComponent()
+ component.bluePrintRuntimeService = bluePrintRuntimeService
+ component.bluePrintClusterService = blueprintClusterService
+
+ runBlocking {
+ component.applyNB(getMockedInput(bluePrintRuntimeService))
+ }
+
+ verify {
+ runBlocking { blueprintClusterService.clusterLock("$lockName@$CDS_LOCK_GROUP") }
+ }
+ verify { runBlocking { clusterLock.unLock() } }
}
/**
@@ -199,10 +294,4 @@ class AbstractComponentFunctionTest {
return executionServiceInput
}
-
- @Test
- fun testComponentScriptExecutorNodeType() {
- val componentScriptExecutor = BluePrintTypes.nodeTypeComponentScriptExecutor()
- assertNotNull(componentScriptExecutor.interfaces, "failed to get interface operations")
- }
}
diff --git a/ms/blueprintsprocessor/modules/services/workflow-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionService.kt b/ms/blueprintsprocessor/modules/services/workflow-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionService.kt
index 6a304e0c7..c703deb37 100644
--- a/ms/blueprintsprocessor/modules/services/workflow-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionService.kt
+++ b/ms/blueprintsprocessor/modules/services/workflow-service/src/main/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionService.kt
@@ -20,6 +20,7 @@ import com.fasterxml.jackson.databind.JsonNode
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceInput
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceOutput
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.StepData
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
import org.onap.ccsdk.cds.blueprintsprocessor.services.execution.AbstractComponentFunction
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
import org.onap.ccsdk.cds.controllerblueprints.core.asJsonPrimitive
@@ -30,7 +31,7 @@ import org.slf4j.LoggerFactory
import org.springframework.stereotype.Service
@Service
-open class NodeTemplateExecutionService {
+open class NodeTemplateExecutionService(private val bluePrintClusterService: BluePrintClusterService) {
private val log = LoggerFactory.getLogger(NodeTemplateExecutionService::class.java)!!
@@ -62,8 +63,9 @@ open class NodeTemplateExecutionService {
// Get the Component Instance
val plugin = BluePrintDependencyService.instance<AbstractComponentFunction>(componentName)
- // Set the Blueprint Service
+ // Set the Blueprint Services
plugin.bluePrintRuntimeService = bluePrintRuntimeService
+ plugin.bluePrintClusterService = bluePrintClusterService
plugin.stepName = nodeTemplateName
// Parent request shouldn't tamper, so need to clone the request and send to the actual component.
diff --git a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BluePrintWorkflowExecutionServiceImplTest.kt b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BluePrintWorkflowExecutionServiceImplTest.kt
index 330056221..90b7200c9 100644
--- a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BluePrintWorkflowExecutionServiceImplTest.kt
+++ b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BluePrintWorkflowExecutionServiceImplTest.kt
@@ -26,6 +26,7 @@ import org.junit.Test
import org.junit.runner.RunWith
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceInput
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceOutput
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
import org.onap.ccsdk.cds.blueprintsprocessor.services.workflow.mock.MockComponentFunction
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintProcessorException
@@ -34,6 +35,7 @@ import org.onap.ccsdk.cds.controllerblueprints.core.service.BluePrintDependencyS
import org.onap.ccsdk.cds.controllerblueprints.core.utils.BluePrintMetadataUtils
import org.onap.ccsdk.cds.controllerblueprints.core.utils.JacksonUtils
import org.springframework.beans.factory.annotation.Autowired
+import org.springframework.boot.test.mock.mockito.MockBean
import org.springframework.test.context.ContextConfiguration
import org.springframework.test.context.junit4.SpringRunner
import kotlin.test.assertEquals
@@ -47,6 +49,9 @@ class BluePrintWorkflowExecutionServiceImplTest {
@Autowired
lateinit var bluePrintWorkflowExecutionService: BluePrintWorkflowExecutionService<ExecutionServiceInput, ExecutionServiceOutput>
+ @MockBean
+ lateinit var bluePrintClusterService: BluePrintClusterService
+
@Before
fun init() {
mockkObject(BluePrintDependencyService)
diff --git a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BlueprintServiceLogicTest.kt b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BlueprintServiceLogicTest.kt
index 3d44894ee..d391050a6 100644
--- a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BlueprintServiceLogicTest.kt
+++ b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/BlueprintServiceLogicTest.kt
@@ -22,6 +22,7 @@ import org.junit.Before
import org.junit.Test
import org.junit.runner.RunWith
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceInput
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
import org.onap.ccsdk.cds.blueprintsprocessor.services.workflow.executor.ComponentExecuteNodeExecutor
import org.onap.ccsdk.cds.blueprintsprocessor.services.workflow.mock.PrototypeComponentFunction
import org.onap.ccsdk.cds.blueprintsprocessor.services.workflow.mock.SingletonComponentFunction
@@ -30,6 +31,7 @@ import org.onap.ccsdk.cds.controllerblueprints.core.service.BluePrintDependencyS
import org.onap.ccsdk.cds.controllerblueprints.core.utils.BluePrintMetadataUtils
import org.onap.ccsdk.cds.controllerblueprints.core.utils.JacksonReactorUtils
import org.springframework.beans.factory.annotation.Autowired
+import org.springframework.boot.test.mock.mockito.MockBean
import org.springframework.context.ApplicationContext
import org.springframework.test.context.ContextConfiguration
import org.springframework.test.context.junit4.SpringRunner
@@ -46,6 +48,9 @@ class BlueprintServiceLogicTest {
@Autowired
lateinit var dgWorkflowExecutionService: DGWorkflowExecutionService
+ @MockBean
+ lateinit var bluePrintClusterService: BluePrintClusterService
+
@Before
fun init() {
BluePrintDependencyService.inject(applicationContext)
diff --git a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/DGWorkflowExecutionServiceTest.kt b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/DGWorkflowExecutionServiceTest.kt
index 86d3be26b..cc1bfee7f 100644
--- a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/DGWorkflowExecutionServiceTest.kt
+++ b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/DGWorkflowExecutionServiceTest.kt
@@ -22,12 +22,14 @@ import org.junit.Before
import org.junit.Test
import org.junit.runner.RunWith
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceInput
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
import org.onap.ccsdk.cds.blueprintsprocessor.services.workflow.executor.ComponentExecuteNodeExecutor
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
import org.onap.ccsdk.cds.controllerblueprints.core.service.BluePrintDependencyService
import org.onap.ccsdk.cds.controllerblueprints.core.utils.BluePrintMetadataUtils
import org.onap.ccsdk.cds.controllerblueprints.core.utils.JacksonReactorUtils
import org.springframework.beans.factory.annotation.Autowired
+import org.springframework.boot.test.mock.mockito.MockBean
import org.springframework.context.ApplicationContext
import org.springframework.test.context.ContextConfiguration
import org.springframework.test.context.junit4.SpringRunner
@@ -44,6 +46,9 @@ class DGWorkflowExecutionServiceTest {
@Autowired
lateinit var dgWorkflowExecutionService: DGWorkflowExecutionService
+ @MockBean
+ lateinit var bluePrintClusterService: BluePrintClusterService
+
@Before
fun init() {
BluePrintDependencyService.inject(applicationContext)
diff --git a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/ImperativeWorkflowExecutionServiceTest.kt b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/ImperativeWorkflowExecutionServiceTest.kt
index 415f11d58..1d4738c8d 100644
--- a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/ImperativeWorkflowExecutionServiceTest.kt
+++ b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/ImperativeWorkflowExecutionServiceTest.kt
@@ -17,6 +17,7 @@
package org.onap.ccsdk.cds.blueprintsprocessor.services.workflow
import io.mockk.every
+import io.mockk.mockk
import io.mockk.mockkObject
import io.mockk.unmockkAll
import kotlinx.coroutines.runBlocking
@@ -118,7 +119,7 @@ class ImperativeWorkflowExecutionServiceTest {
ExecutionServiceInput::class.java
)!!
- val bluePrintWorkFlowService = ImperativeBluePrintWorkflowService(NodeTemplateExecutionService())
+ val bluePrintWorkFlowService = ImperativeBluePrintWorkflowService(NodeTemplateExecutionService(mockk()))
val imperativeWorkflowExecutionService = ImperativeWorkflowExecutionService(bluePrintWorkFlowService)
val output = imperativeWorkflowExecutionService
.executeBluePrintWorkflow(bluePrintRuntimeService, executionServiceInput, hashMapOf())
diff --git a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionServiceTest.kt b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionServiceTest.kt
index 1f51a6aae..f30086596 100644
--- a/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionServiceTest.kt
+++ b/ms/blueprintsprocessor/modules/services/workflow-service/src/test/kotlin/org/onap/ccsdk/cds/blueprintsprocessor/services/workflow/NodeTemplateExecutionServiceTest.kt
@@ -24,11 +24,13 @@ import org.junit.Before
import org.junit.Test
import org.junit.runner.RunWith
import org.onap.ccsdk.cds.blueprintsprocessor.core.api.data.ExecutionServiceInput
+import org.onap.ccsdk.cds.blueprintsprocessor.core.service.BluePrintClusterService
import org.onap.ccsdk.cds.blueprintsprocessor.services.workflow.mock.MockComponentFunction
import org.onap.ccsdk.cds.controllerblueprints.core.BluePrintConstants
import org.onap.ccsdk.cds.controllerblueprints.core.service.BluePrintDependencyService
import org.onap.ccsdk.cds.controllerblueprints.core.utils.BluePrintMetadataUtils
import org.onap.ccsdk.cds.controllerblueprints.core.utils.JacksonUtils
+import org.springframework.boot.test.mock.mockito.MockBean
import org.springframework.test.context.ContextConfiguration
import org.springframework.test.context.junit4.SpringRunner
import kotlin.test.assertEquals
@@ -39,6 +41,9 @@ import kotlin.test.assertNotNull
class NodeTemplateExecutionServiceTest {
+ @MockBean
+ lateinit var bluePrintClusterService: BluePrintClusterService
+
@Before
fun init() {
mockkObject(BluePrintDependencyService)
@@ -68,7 +73,7 @@ class NodeTemplateExecutionServiceTest {
bluePrintRuntimeService.assignWorkflowInputs("resource-assignment", input)
val nodeTemplate = "resource-assignment"
- val nodeTemplateExecutionService = NodeTemplateExecutionService()
+ val nodeTemplateExecutionService = NodeTemplateExecutionService(bluePrintClusterService)
val executionServiceOutput = nodeTemplateExecutionService
.executeNodeTemplate(bluePrintRuntimeService, nodeTemplate, executionServiceInput)