You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@nifi.apache.org by GitBox <gi...@apache.org> on 2022/08/09 16:08:11 UTC

[GitHub] [nifi] rliszli opened a new pull request, #6281: NIFI-10312 - Fix MiNiFi C2 integration

rliszli opened a new pull request, #6281:
URL: https://github.com/apache/nifi/pull/6281

   <!-- Licensed to the Apache Software Foundation (ASF) under one or more -->
   <!-- contributor license agreements.  See the NOTICE file distributed with -->
   <!-- this work for additional information regarding copyright ownership. -->
   <!-- The ASF licenses this file to You 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. -->
   
   # Summary
   
   This change fix the communication between the MiNiFi and the C2 Service trough C2 protocol. Also covered the heartbeating and flow update with integration tests.
   
   [NIFI-10312](https://issues.apache.org/jira/browse/NIFI-10312)
   
   # Tracking
   
   Please complete the following tracking steps prior to pull request creation.
   
   ### Issue Tracking
   
   - [ ] [Apache NiFi Jira](https://issues.apache.org/jira/browse/NIFI) issue created
   
   ### Pull Request Tracking
   
   - [ ] Pull Request title starts with Apache NiFi Jira issue number, such as `NIFI-00000`
   - [ ] Pull Request commit message starts with Apache NiFi Jira issue number, as such `NIFI-00000`
   
   ### Pull Request Formatting
   
   - [ ] Pull Request based on current revision of the `main` branch
   - [ ] Pull Request refers to a feature branch with one commit containing changes
   
   # Verification
   
   Please indicate the verification steps performed prior to pull request creation.
   
   ### Build
   
   - [ ] Build completed using `mvn clean install -P contrib-check`
     - [ ] JDK 8
     - [ ] JDK 11
     - [ ] JDK 17
   
   ### Licensing
   
   - [ ] New dependencies are compatible with the [Apache License 2.0](https://apache.org/licenses/LICENSE-2.0) according to the [License Policy](https://www.apache.org/legal/resolved.html)
   - [ ] New dependencies are documented in applicable `LICENSE` and `NOTICE` files
   
   ### Documentation
   
   - [ ] Documentation formatting appears as expected in rendered files
   


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948701968


##########
c2/c2-client-bundle/c2-client-service/src/test/java/org/apache/nifi/c2/client/service/operation/UpdateConfigurationOperationHandlerTest.java:
##########
@@ -17,13 +17,12 @@
 package org.apache.nifi.c2.client.service.operation;
 
 import static org.apache.commons.lang3.StringUtils.EMPTY;
-import static org.apache.nifi.c2.client.service.operation.UpdateConfigurationOperationHandler.LOCATION;

Review Comment:
   Sure, done.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] bejancsaba commented on pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
bejancsaba commented on PR #6281:
URL: https://github.com/apache/nifi/pull/6281#issuecomment-1228340726

   Thank you for the changes my approve was not removed from the PR so still +1 from my side.


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r954605703


##########
c2/c2-protocol/c2-protocol-api/pom.xml:
##########
@@ -33,5 +33,9 @@ limitations under the License.
             <artifactId>c2-protocol-component-api</artifactId>
             <version>1.18.0-SNAPSHOT</version>
         </dependency>
+        <dependency>
+            <groupId>com.fasterxml.jackson.core</groupId>

Review Comment:
   Removed the dependency and added a custom deserializer.



##########
c2/c2-protocol/c2-protocol-api/src/main/java/org/apache/nifi/c2/protocol/api/OperandType.java:
##########
@@ -28,6 +30,7 @@ public enum OperandType {
     MANIFEST,
     REPOSITORY;
 
+    @JsonCreator

Review Comment:
   Removed.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948703762


##########
minifi/minifi-integration-tests/src/test/java/org/apache/nifi/minifi/integration/c2/C2ProtocolIntegrationTest.java:
##########
@@ -0,0 +1,114 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You 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.apache.nifi.minifi.integration.c2;
+
+import com.palantir.docker.compose.DockerComposeExtension;
+import com.palantir.docker.compose.connection.waiting.HealthChecks;
+import org.apache.nifi.minifi.integration.util.LogUtil;
+import org.apache.nifi.security.util.KeystoreType;
+import org.apache.nifi.security.util.SslContextFactory;
+import org.apache.nifi.security.util.StandardTlsConfiguration;
+import org.apache.nifi.security.util.TlsConfiguration;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandalone;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandaloneCommandLine;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.Test;
+import org.junit.jupiter.api.Timeout;
+
+import javax.net.ssl.SSLContext;
+import javax.net.ssl.SSLSocketFactory;
+import java.nio.file.Files;
+import java.nio.file.Path;
+import java.nio.file.Paths;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.List;
+
+@Timeout(60)
+public class C2ProtocolIntegrationTest {
+    private static final String AGENT_1 = "minifi-edge1";
+    private static final String AGENT_2 = "minifi-edge2";
+    private static final String AGENT_CLASS = "raspi3";
+    private static final String SERVICE = "c2-authoritative";
+    private static final String CONFIG_YAML = "config.text.yml.v2";
+    private static Path certificatesDirectory;
+    private static SSLContext trustSslContext;
+    private static SSLSocketFactory healthCheckSocketFactory;
+    public static DockerComposeExtension docker = DockerComposeExtension.builder()
+            .file("target/test-classes/docker-compose-c2-protocol.yml")
+            .waitingForService(AGENT_1, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .waitingForService(AGENT_2, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .build();
+
+    private static Path resourceDirectory;
+    private static Path authoritativeFiles;
+    private static Path minifiEdge1Version2;
+    private static Path minifiEdge2Version2;
+
+    /**
+     * Generates certificates with the tls-toolkit and then starts up the docker compose file
+     */
+    @BeforeAll
+    public static void init() throws Exception {
+        resourceDirectory = Paths.get(C2ProtocolIntegrationTest.class.getClassLoader()
+                .getResource("docker-compose-c2-protocol.yml").getFile()).getParent();
+        certificatesDirectory = resourceDirectory.toAbsolutePath().resolve("certificates-c2-protocol");
+        authoritativeFiles = resourceDirectory.resolve("c2").resolve("protocol").resolve(SERVICE).resolve("files");
+        minifiEdge1Version2 = authoritativeFiles.resolve("edge1").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+        minifiEdge2Version2 = authoritativeFiles.resolve("edge2").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+
+        if (Files.exists(minifiEdge1Version2)) {
+            Files.delete(minifiEdge1Version2);
+        }
+        if (Files.exists(minifiEdge2Version2)) {
+            Files.delete(minifiEdge2Version2);
+        }
+
+        List<String> toolkitCommandLine = new ArrayList<>(Arrays.asList("-O", "-o", certificatesDirectory.toFile().getAbsolutePath(), "-S", "badKeystorePass", "-P", "badTrustPass"));
+        for (String serverHostname : Arrays.asList(SERVICE, AGENT_1, AGENT_2)) {
+            toolkitCommandLine.add("-n");
+            toolkitCommandLine.add(serverHostname);
+        }
+        Files.createDirectories(certificatesDirectory);
+        TlsToolkitStandaloneCommandLine tlsToolkitStandaloneCommandLine = new TlsToolkitStandaloneCommandLine();
+        tlsToolkitStandaloneCommandLine.parse(toolkitCommandLine.toArray(new String[toolkitCommandLine.size()]));
+        new TlsToolkitStandalone().createNifiKeystoresAndTrustStores(tlsToolkitStandaloneCommandLine.createConfig());
+
+        TlsConfiguration tlsConfiguration = new StandardTlsConfiguration(
+                null,null,null,
+                certificatesDirectory.resolve(SERVICE).resolve("truststore.jks").toFile().getAbsolutePath(),
+                "badTrustPass",
+                KeystoreType.JKS);
+        trustSslContext = SslContextFactory.createSslContext(tlsConfiguration);
+        healthCheckSocketFactory = trustSslContext.getSocketFactory();
+
+        docker.before();
+    }
+
+    @AfterAll
+    public static void stopDocker() {
+        docker.after();
+    }
+
+    @Test
+    public void testMiNiFiHeartbeat() throws Exception {

Review Comment:
   I did the rename, thanks. The goal with this test was to test that all the agents recives the updates. If there would be 2 agents with different classes, it would test the same thing, that one agent gets the update, twice. 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] bejancsaba commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
bejancsaba commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948729909


##########
minifi/minifi-integration-tests/src/test/java/org/apache/nifi/minifi/integration/c2/C2ProtocolIntegrationTest.java:
##########
@@ -0,0 +1,114 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You 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.apache.nifi.minifi.integration.c2;
+
+import com.palantir.docker.compose.DockerComposeExtension;
+import com.palantir.docker.compose.connection.waiting.HealthChecks;
+import org.apache.nifi.minifi.integration.util.LogUtil;
+import org.apache.nifi.security.util.KeystoreType;
+import org.apache.nifi.security.util.SslContextFactory;
+import org.apache.nifi.security.util.StandardTlsConfiguration;
+import org.apache.nifi.security.util.TlsConfiguration;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandalone;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandaloneCommandLine;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.Test;
+import org.junit.jupiter.api.Timeout;
+
+import javax.net.ssl.SSLContext;
+import javax.net.ssl.SSLSocketFactory;
+import java.nio.file.Files;
+import java.nio.file.Path;
+import java.nio.file.Paths;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.List;
+
+@Timeout(60)
+public class C2ProtocolIntegrationTest {
+    private static final String AGENT_1 = "minifi-edge1";
+    private static final String AGENT_2 = "minifi-edge2";
+    private static final String AGENT_CLASS = "raspi3";
+    private static final String SERVICE = "c2-authoritative";
+    private static final String CONFIG_YAML = "config.text.yml.v2";
+    private static Path certificatesDirectory;
+    private static SSLContext trustSslContext;
+    private static SSLSocketFactory healthCheckSocketFactory;
+    public static DockerComposeExtension docker = DockerComposeExtension.builder()
+            .file("target/test-classes/docker-compose-c2-protocol.yml")
+            .waitingForService(AGENT_1, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .waitingForService(AGENT_2, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .build();
+
+    private static Path resourceDirectory;
+    private static Path authoritativeFiles;
+    private static Path minifiEdge1Version2;
+    private static Path minifiEdge2Version2;
+
+    /**
+     * Generates certificates with the tls-toolkit and then starts up the docker compose file
+     */
+    @BeforeAll
+    public static void init() throws Exception {
+        resourceDirectory = Paths.get(C2ProtocolIntegrationTest.class.getClassLoader()
+                .getResource("docker-compose-c2-protocol.yml").getFile()).getParent();
+        certificatesDirectory = resourceDirectory.toAbsolutePath().resolve("certificates-c2-protocol");
+        authoritativeFiles = resourceDirectory.resolve("c2").resolve("protocol").resolve(SERVICE).resolve("files");
+        minifiEdge1Version2 = authoritativeFiles.resolve("edge1").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+        minifiEdge2Version2 = authoritativeFiles.resolve("edge2").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+
+        if (Files.exists(minifiEdge1Version2)) {
+            Files.delete(minifiEdge1Version2);
+        }
+        if (Files.exists(minifiEdge2Version2)) {
+            Files.delete(minifiEdge2Version2);
+        }
+
+        List<String> toolkitCommandLine = new ArrayList<>(Arrays.asList("-O", "-o", certificatesDirectory.toFile().getAbsolutePath(), "-S", "badKeystorePass", "-P", "badTrustPass"));
+        for (String serverHostname : Arrays.asList(SERVICE, AGENT_1, AGENT_2)) {
+            toolkitCommandLine.add("-n");
+            toolkitCommandLine.add(serverHostname);
+        }
+        Files.createDirectories(certificatesDirectory);
+        TlsToolkitStandaloneCommandLine tlsToolkitStandaloneCommandLine = new TlsToolkitStandaloneCommandLine();
+        tlsToolkitStandaloneCommandLine.parse(toolkitCommandLine.toArray(new String[toolkitCommandLine.size()]));
+        new TlsToolkitStandalone().createNifiKeystoresAndTrustStores(tlsToolkitStandaloneCommandLine.createConfig());
+
+        TlsConfiguration tlsConfiguration = new StandardTlsConfiguration(
+                null,null,null,
+                certificatesDirectory.resolve(SERVICE).resolve("truststore.jks").toFile().getAbsolutePath(),
+                "badTrustPass",
+                KeystoreType.JKS);
+        trustSslContext = SslContextFactory.createSslContext(tlsConfiguration);
+        healthCheckSocketFactory = trustSslContext.getSocketFactory();
+
+        docker.before();
+    }
+
+    @AfterAll
+    public static void stopDocker() {
+        docker.after();
+    }
+
+    @Test
+    public void testMiNiFiHeartbeat() throws Exception {

Review Comment:
   Maybe I misunderstand something. I understand that it is good to test that if there are multiple agents under the same class they both download the flow. Let's leave it as is but I would also add an additional agent class and validate that an agent under a different agent class gets a different flow. These integration tests should validate integration between agent and minifi-c2-servive and the scenario when agents are under different classes from agent point of view is the samee but it is wildly different from the service point of view so adding such test would be really important.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] bejancsaba commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
bejancsaba commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948724754


##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/C2ClientService.java:
##########
@@ -41,8 +41,12 @@ public C2ClientService(C2Client client, C2HeartbeatFactory c2HeartbeatFactory, C
     }
 
     public void sendHeartbeat(RuntimeInfoWrapper runtimeInfoWrapper) {
-        C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
-        client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        try {
+            C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
+            client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        } catch (Exception e) {

Review Comment:
   I'm  not sure what would be the wrong input but I think it should be handled on the clint side (or processing side if it came from there) I would remove this exception catching from here and handle the issue where it potentially comes from.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948702984


##########
minifi/minifi-integration-tests/src/test/resources/logback.xml:
##########
@@ -55,7 +55,7 @@
     
     <!-- valid logging levels: TRACE, DEBUG, INFO, WARN, ERROR -->
     
-    <logger name="org.apache.nifi" level="INFO"/>
+    <logger name="org.apache.nifi" level="DEBUG"/>

Review Comment:
   Thanks, reverted.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] exceptionfactory closed pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
exceptionfactory closed pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration
URL: https://github.com/apache/nifi/pull/6281


-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] bejancsaba commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
bejancsaba commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r947718839


##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/C2ClientService.java:
##########
@@ -41,8 +41,12 @@ public C2ClientService(C2Client client, C2HeartbeatFactory c2HeartbeatFactory, C
     }
 
     public void sendHeartbeat(RuntimeInfoWrapper runtimeInfoWrapper) {
-        C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
-        client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        try {
+            C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
+            client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        } catch (Exception e) {

Review Comment:
   I think the issues should be handled on the client side there is already logic to cover IOException when sending heartbeat with logging for  "Send Heartbeat failed [{}]" if there are cases when other exception is expected that should be covered there in my opinion. What do you think?



##########
c2/c2-client-bundle/c2-client-service/src/test/java/org/apache/nifi/c2/client/service/operation/UpdateConfigurationOperationHandlerTest.java:
##########
@@ -62,12 +61,34 @@ void testUpdateConfigurationOperationHandlerCreateSuccess() {
     }
 
     @Test
-    void testHandleThrowsExceptionForIncorrectArg() {
+    void testHandleIncorrectArg() {
         UpdateConfigurationOperationHandler handler = new UpdateConfigurationOperationHandler(null, null, null);
         C2Operation operation = new C2Operation();
         operation.setArgs(INCORRECT_LOCATION_MAP);
 
-        IllegalStateException exception = assertThrows(IllegalStateException.class, () -> handler.handle(operation));
+        C2OperationAck response = handler.handle(operation);
+
+        assertEquals(C2OperationState.OperationState.NOT_APPLIED, response.getOperationState().getState());
+    }
+
+    @Test
+    void testHandleFlowIdInArg() {
+        Function<byte[], Boolean> successUpdate = x -> true;
+        when(flowIdHolder.getFlowId()).thenReturn(FLOW_ID);
+        when(client.retrieveUpdateContent(any())).thenReturn(Optional.of("content".getBytes()));
+        UpdateConfigurationOperationHandler handler = new UpdateConfigurationOperationHandler(client, flowIdHolder, successUpdate);
+        C2Operation operation = new C2Operation();
+        operation.setIdentifier(OPERATION_ID);
+
+        Map<String, String> args = new HashMap<>();
+        args.putAll(INCORRECT_LOCATION_MAP);
+        args.put("flowId", "argsFlowId");

Review Comment:
   Please use the FLOW_ID constant defined in the class under test. That way we can avoid test fragility in case of later changes.



##########
minifi/minifi-integration-tests/src/test/resources/logback.xml:
##########
@@ -55,7 +55,7 @@
     
     <!-- valid logging levels: TRACE, DEBUG, INFO, WARN, ERROR -->
     
-    <logger name="org.apache.nifi" level="INFO"/>
+    <logger name="org.apache.nifi" level="DEBUG"/>

Review Comment:
   I understand it is useful for investigation but is it actually needed to be changed permanently. My guess is that it makes the log really noisy.



##########
c2/c2-client-bundle/c2-client-service/src/test/java/org/apache/nifi/c2/client/service/operation/UpdateConfigurationOperationHandlerTest.java:
##########
@@ -17,13 +17,12 @@
 package org.apache.nifi.c2.client.service.operation;
 
 import static org.apache.commons.lang3.StringUtils.EMPTY;
-import static org.apache.nifi.c2.client.service.operation.UpdateConfigurationOperationHandler.LOCATION;

Review Comment:
   This was intentionally not private in the class so it can be used in the test without duplication. Can you revert back that change?



##########
c2/c2-protocol/c2-protocol-api/pom.xml:
##########
@@ -33,5 +33,9 @@ limitations under the License.
             <artifactId>c2-protocol-component-api</artifactId>
             <version>1.18.0-SNAPSHOT</version>
         </dependency>
+        <dependency>
+            <groupId>com.fasterxml.jackson.core</groupId>

Review Comment:
   I think we should avoid polluting the -api modules. They should be pure only with the models (maybe it will be used with different serialisation not Jackson somewhere). Could you look into a different way of addressing the issue this solves? We can discuss separately if needed.



##########
c2/c2-protocol/c2-protocol-api/src/main/java/org/apache/nifi/c2/protocol/api/OperandType.java:
##########
@@ -28,6 +30,7 @@ public enum OperandType {
     MANIFEST,
     REPOSITORY;
 
+    @JsonCreator

Review Comment:
   Please see my comment above.



##########
minifi/minifi-integration-tests/src/test/java/org/apache/nifi/minifi/integration/c2/C2ProtocolIntegrationTest.java:
##########
@@ -0,0 +1,114 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You 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.apache.nifi.minifi.integration.c2;
+
+import com.palantir.docker.compose.DockerComposeExtension;
+import com.palantir.docker.compose.connection.waiting.HealthChecks;
+import org.apache.nifi.minifi.integration.util.LogUtil;
+import org.apache.nifi.security.util.KeystoreType;
+import org.apache.nifi.security.util.SslContextFactory;
+import org.apache.nifi.security.util.StandardTlsConfiguration;
+import org.apache.nifi.security.util.TlsConfiguration;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandalone;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandaloneCommandLine;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.Test;
+import org.junit.jupiter.api.Timeout;
+
+import javax.net.ssl.SSLContext;
+import javax.net.ssl.SSLSocketFactory;
+import java.nio.file.Files;
+import java.nio.file.Path;
+import java.nio.file.Paths;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.List;
+
+@Timeout(60)
+public class C2ProtocolIntegrationTest {
+    private static final String AGENT_1 = "minifi-edge1";
+    private static final String AGENT_2 = "minifi-edge2";
+    private static final String AGENT_CLASS = "raspi3";
+    private static final String SERVICE = "c2-authoritative";
+    private static final String CONFIG_YAML = "config.text.yml.v2";
+    private static Path certificatesDirectory;
+    private static SSLContext trustSslContext;
+    private static SSLSocketFactory healthCheckSocketFactory;
+    public static DockerComposeExtension docker = DockerComposeExtension.builder()
+            .file("target/test-classes/docker-compose-c2-protocol.yml")
+            .waitingForService(AGENT_1, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .waitingForService(AGENT_2, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .build();
+
+    private static Path resourceDirectory;
+    private static Path authoritativeFiles;
+    private static Path minifiEdge1Version2;
+    private static Path minifiEdge2Version2;
+
+    /**
+     * Generates certificates with the tls-toolkit and then starts up the docker compose file
+     */
+    @BeforeAll
+    public static void init() throws Exception {
+        resourceDirectory = Paths.get(C2ProtocolIntegrationTest.class.getClassLoader()
+                .getResource("docker-compose-c2-protocol.yml").getFile()).getParent();
+        certificatesDirectory = resourceDirectory.toAbsolutePath().resolve("certificates-c2-protocol");
+        authoritativeFiles = resourceDirectory.resolve("c2").resolve("protocol").resolve(SERVICE).resolve("files");
+        minifiEdge1Version2 = authoritativeFiles.resolve("edge1").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+        minifiEdge2Version2 = authoritativeFiles.resolve("edge2").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+
+        if (Files.exists(minifiEdge1Version2)) {
+            Files.delete(minifiEdge1Version2);
+        }
+        if (Files.exists(minifiEdge2Version2)) {
+            Files.delete(minifiEdge2Version2);
+        }
+
+        List<String> toolkitCommandLine = new ArrayList<>(Arrays.asList("-O", "-o", certificatesDirectory.toFile().getAbsolutePath(), "-S", "badKeystorePass", "-P", "badTrustPass"));
+        for (String serverHostname : Arrays.asList(SERVICE, AGENT_1, AGENT_2)) {
+            toolkitCommandLine.add("-n");
+            toolkitCommandLine.add(serverHostname);
+        }
+        Files.createDirectories(certificatesDirectory);
+        TlsToolkitStandaloneCommandLine tlsToolkitStandaloneCommandLine = new TlsToolkitStandaloneCommandLine();
+        tlsToolkitStandaloneCommandLine.parse(toolkitCommandLine.toArray(new String[toolkitCommandLine.size()]));
+        new TlsToolkitStandalone().createNifiKeystoresAndTrustStores(tlsToolkitStandaloneCommandLine.createConfig());
+
+        TlsConfiguration tlsConfiguration = new StandardTlsConfiguration(
+                null,null,null,
+                certificatesDirectory.resolve(SERVICE).resolve("truststore.jks").toFile().getAbsolutePath(),
+                "badTrustPass",
+                KeystoreType.JKS);
+        trustSslContext = SslContextFactory.createSslContext(tlsConfiguration);
+        healthCheckSocketFactory = trustSslContext.getSocketFactory();
+
+        docker.before();
+    }
+
+    @AfterAll
+    public static void stopDocker() {
+        docker.after();
+    }
+
+    @Test
+    public void testMiNiFiHeartbeat() throws Exception {

Review Comment:
   Based on the expected.json this validates not only heartbeating but flow definition download as well. Could you please rename the test to reflect that?
   Can we tweak the test to have the two agents belonging to different classes? That would be  amore realistic test in my opinion. What do you think?



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948702181


##########
c2/c2-client-bundle/c2-client-service/src/test/java/org/apache/nifi/c2/client/service/operation/UpdateConfigurationOperationHandlerTest.java:
##########
@@ -62,12 +61,34 @@ void testUpdateConfigurationOperationHandlerCreateSuccess() {
     }
 
     @Test
-    void testHandleThrowsExceptionForIncorrectArg() {
+    void testHandleIncorrectArg() {
         UpdateConfigurationOperationHandler handler = new UpdateConfigurationOperationHandler(null, null, null);
         C2Operation operation = new C2Operation();
         operation.setArgs(INCORRECT_LOCATION_MAP);
 
-        IllegalStateException exception = assertThrows(IllegalStateException.class, () -> handler.handle(operation));
+        C2OperationAck response = handler.handle(operation);
+
+        assertEquals(C2OperationState.OperationState.NOT_APPLIED, response.getOperationState().getState());
+    }
+
+    @Test
+    void testHandleFlowIdInArg() {
+        Function<byte[], Boolean> successUpdate = x -> true;
+        when(flowIdHolder.getFlowId()).thenReturn(FLOW_ID);
+        when(client.retrieveUpdateContent(any())).thenReturn(Optional.of("content".getBytes()));
+        UpdateConfigurationOperationHandler handler = new UpdateConfigurationOperationHandler(client, flowIdHolder, successUpdate);
+        C2Operation operation = new C2Operation();
+        operation.setIdentifier(OPERATION_ID);
+
+        Map<String, String> args = new HashMap<>();
+        args.putAll(INCORRECT_LOCATION_MAP);
+        args.put("flowId", "argsFlowId");

Review Comment:
   Done.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] bejancsaba commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
bejancsaba commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948726341


##########
c2/c2-protocol/c2-protocol-api/pom.xml:
##########
@@ -33,5 +33,9 @@ limitations under the License.
             <artifactId>c2-protocol-component-api</artifactId>
             <version>1.18.0-SNAPSHOT</version>
         </dependency>
+        <dependency>
+            <groupId>com.fasterxml.jackson.core</groupId>

Review Comment:
   I understand the reason but we have to keep the api module clean as this is basically the c2 protocol domain definition which shouldn't be opinionated (if someone is using the protocol but uses different things for serialisation they shouldn't depend on jackson). So please remove it from here.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] exceptionfactory commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
exceptionfactory commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r949069069


##########
c2/c2-protocol/c2-protocol-api/pom.xml:
##########
@@ -33,5 +33,9 @@ limitations under the License.
             <artifactId>c2-protocol-component-api</artifactId>
             <version>1.18.0-SNAPSHOT</version>
         </dependency>
+        <dependency>
+            <groupId>com.fasterxml.jackson.core</groupId>

Review Comment:
   Concurring with @bejancsaba, although the Jackson annotations are convenient, the `c2-protocol-api` dependencies should be kept at a minimum, so writing a custom deserializer would be preferred in this particular case.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] exceptionfactory commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
exceptionfactory commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r956165997


##########
minifi/minifi-c2/minifi-c2-service/src/main/java/org/apache/nifi/minifi/c2/service/ConfigService.java:
##########
@@ -229,7 +229,7 @@ public Response heartbeat(
                     configuration = configurationProviderValue.getConfiguration();
                 } catch (ConfigurationProviderException cpe) {
                     logger.warn("No flow available for agent class " + agentClass + ", returning No Content (204)");
-                    response = Response.noContent().build();
+                    response = Response.ok(new C2HeartbeatResponse()).build();

Review Comment:
   The warning indicates that an HTTP 204 will be returned, but the change returns an empty heartbeat response with an HTTP 200. It seems better in general to return the HTTP 204 if there is no response, but if there is a particular reason for return an empty heartbeat response, the warning log should be adjusted.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] bejancsaba commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
bejancsaba commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r955386864


##########
c2/c2-protocol/c2-protocol-api/pom.xml:
##########
@@ -33,5 +33,9 @@ limitations under the License.
             <artifactId>c2-protocol-component-api</artifactId>
             <version>1.18.0-SNAPSHOT</version>
         </dependency>
+        <dependency>
+            <groupId>com.fasterxml.jackson.core</groupId>

Review Comment:
   Thank you!



##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/C2ClientService.java:
##########
@@ -41,8 +41,8 @@ public C2ClientService(C2Client client, C2HeartbeatFactory c2HeartbeatFactory, C
     }
 
     public void sendHeartbeat(RuntimeInfoWrapper runtimeInfoWrapper) {
-        C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
-        client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+            C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);

Review Comment:
   This indentation seems unnecessary could you please double check?



##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/operation/UpdateConfigurationOperationHandler.java:
##########
@@ -75,30 +75,44 @@ public C2OperationAck handle(C2Operation operation) {
             .map(map -> map.get(LOCATION))
             .orElse(EMPTY);
 
-        String newFlowId = parseFlowId(updateLocation);
-        if (flowIdHolder.getFlowId() == null || !flowIdHolder.getFlowId().equals(newFlowId)) {
-            logger.info("Will perform flow update from {} for operation #{}. Previous flow id was {}, replacing with new id {}", updateLocation, opIdentifier,
-                flowIdHolder.getFlowId() == null ? "not set" : flowIdHolder.getFlowId(), newFlowId);
+        String flowId = getFlowId(operation.getArgs(), updateLocation);
+        if (flowId == null) {
+            state.setState(C2OperationState.OperationState.NOT_APPLIED);
+            state.setDetails("Could not get flowId from the operation.");
+            logger.info("FlowId is missing, no update will be performed.");
         } else {
-            logger.info("Flow is current, no update is necessary...");
+            if (flowIdHolder.getFlowId() == null || !flowIdHolder.getFlowId().equals(flowId)) {
+                logger.info("Will perform flow update from {} for operation #{}. Previous flow id was {}, replacing with new id {}", updateLocation, opIdentifier,
+                        flowIdHolder.getFlowId() == null ? "not set" : flowIdHolder.getFlowId(), flowId);
+            } else {
+                logger.info("Flow is current, no update is necessary...");
+            }
+            flowIdHolder.setFlowId(flowId);
+            state.setState(updateFlow(opIdentifier, updateLocation));
         }
+        return operationAck;
+    }
 
-        flowIdHolder.setFlowId(newFlowId);
+    private C2OperationState.OperationState updateFlow(String opIdentifier, String updateLocation) {
         Optional<byte[]> updateContent = client.retrieveUpdateContent(updateLocation);
         if (updateContent.isPresent()) {
             if (updateFlow.apply(updateContent.get())) {
-                state.setState(C2OperationState.OperationState.FULLY_APPLIED);
                 logger.debug("Update configuration applied for operation #{}.", opIdentifier);
+                return C2OperationState.OperationState.FULLY_APPLIED;
             } else {
-                state.setState(C2OperationState.OperationState.NOT_APPLIED);
                 logger.error("Update resulted in error for operation #{}.", opIdentifier);
+                return C2OperationState.OperationState.NOT_APPLIED;
             }
         } else {
-            state.setState(C2OperationState.OperationState.NOT_APPLIED);
             logger.error("Update content retrieval resulted in empty content so flow update was omitted for operation #{}.", opIdentifier);
+            return C2OperationState.OperationState.NOT_APPLIED;
         }
+    }
 
-        return operationAck;
+    private String getFlowId(Map<String, String> args, String updateLocation) {
+        Optional<String> flowId = Optional.ofNullable(args)

Review Comment:
   You don't need this local varibale you could simply chain further
   ```
   return Optional.ofNullable(args)
   .map(map -> map.get(FLOW_ID))
   .orElseGet(() -> parseFlowId(updateLocation));
   ```
   What do you think?



##########
minifi/minifi-integration-tests/src/test/resources/c2/protocol/minifi-edge3/expected.json:
##########
@@ -0,0 +1,17 @@
+[
+  {
+    "pattern": "200 OK https://c2-authoritative:10443/c2/config/heartbeat"
+  },
+  {
+    "pattern": "\"operation\":\"UPDATE\",\"operand\":\"CONFIGURATION\""
+  },
+  {
+    "pattern": "200 OK https://c2-authoritative:10443/c2/config\\?class=raspi4"
+  },
+  {
+    "pattern": "200 OK https://c2-authoritative:10443/c2/config/acknowledge"
+  },
+  {
+    "pattern": "^__testTextRaspi4__$"

Review Comment:
   Thanks for this, I know it wasn't straightforward but this is really important that now we are validating the parallel handling / publishing of different agent classes.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r956980142


##########
minifi/minifi-c2/minifi-c2-service/src/main/java/org/apache/nifi/minifi/c2/service/ConfigService.java:
##########
@@ -229,7 +229,7 @@ public Response heartbeat(
                     configuration = configurationProviderValue.getConfiguration();
                 } catch (ConfigurationProviderException cpe) {
                     logger.warn("No flow available for agent class " + agentClass + ", returning No Content (204)");
-                    response = Response.noContent().build();
+                    response = Response.ok(new C2HeartbeatResponse()).build();

Review Comment:
   Nice catch, thank you. Did this change because the agents cannot handle the 204 responses with empty body. Updated the warning message accordingly.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948705878


##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/C2ClientService.java:
##########
@@ -41,8 +41,12 @@ public C2ClientService(C2Client client, C2HeartbeatFactory c2HeartbeatFactory, C
     }
 
     public void sendHeartbeat(RuntimeInfoWrapper runtimeInfoWrapper) {
-        C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
-        client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        try {
+            C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
+            client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        } catch (Exception e) {

Review Comment:
   This was added as a safety net. Without it, if an unhandled exception thrown, the heartbeating stops. During the local development I run into this 2 times. One from the deserialization in C2HttpClient.sendHeartbeat() and one from the UpdateConfigurationOperationHandler.  The later got fixed locally as you suggested.  The 1st happened due to wrong input which, I think, can happen. If you still think it should be removed, let me know and I'll remove it.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r954604435


##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/C2ClientService.java:
##########
@@ -41,8 +41,12 @@ public C2ClientService(C2Client client, C2HeartbeatFactory c2HeartbeatFactory, C
     }
 
     public void sendHeartbeat(RuntimeInfoWrapper runtimeInfoWrapper) {
-        C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
-        client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        try {
+            C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
+            client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+        } catch (Exception e) {

Review Comment:
   Removed.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r955751469


##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/operation/UpdateConfigurationOperationHandler.java:
##########
@@ -75,30 +75,44 @@ public C2OperationAck handle(C2Operation operation) {
             .map(map -> map.get(LOCATION))
             .orElse(EMPTY);
 
-        String newFlowId = parseFlowId(updateLocation);
-        if (flowIdHolder.getFlowId() == null || !flowIdHolder.getFlowId().equals(newFlowId)) {
-            logger.info("Will perform flow update from {} for operation #{}. Previous flow id was {}, replacing with new id {}", updateLocation, opIdentifier,
-                flowIdHolder.getFlowId() == null ? "not set" : flowIdHolder.getFlowId(), newFlowId);
+        String flowId = getFlowId(operation.getArgs(), updateLocation);
+        if (flowId == null) {
+            state.setState(C2OperationState.OperationState.NOT_APPLIED);
+            state.setDetails("Could not get flowId from the operation.");
+            logger.info("FlowId is missing, no update will be performed.");
         } else {
-            logger.info("Flow is current, no update is necessary...");
+            if (flowIdHolder.getFlowId() == null || !flowIdHolder.getFlowId().equals(flowId)) {
+                logger.info("Will perform flow update from {} for operation #{}. Previous flow id was {}, replacing with new id {}", updateLocation, opIdentifier,
+                        flowIdHolder.getFlowId() == null ? "not set" : flowIdHolder.getFlowId(), flowId);
+            } else {
+                logger.info("Flow is current, no update is necessary...");
+            }
+            flowIdHolder.setFlowId(flowId);
+            state.setState(updateFlow(opIdentifier, updateLocation));
         }
+        return operationAck;
+    }
 
-        flowIdHolder.setFlowId(newFlowId);
+    private C2OperationState.OperationState updateFlow(String opIdentifier, String updateLocation) {
         Optional<byte[]> updateContent = client.retrieveUpdateContent(updateLocation);
         if (updateContent.isPresent()) {
             if (updateFlow.apply(updateContent.get())) {
-                state.setState(C2OperationState.OperationState.FULLY_APPLIED);
                 logger.debug("Update configuration applied for operation #{}.", opIdentifier);
+                return C2OperationState.OperationState.FULLY_APPLIED;
             } else {
-                state.setState(C2OperationState.OperationState.NOT_APPLIED);
                 logger.error("Update resulted in error for operation #{}.", opIdentifier);
+                return C2OperationState.OperationState.NOT_APPLIED;
             }
         } else {
-            state.setState(C2OperationState.OperationState.NOT_APPLIED);
             logger.error("Update content retrieval resulted in empty content so flow update was omitted for operation #{}.", opIdentifier);
+            return C2OperationState.OperationState.NOT_APPLIED;
         }
+    }
 
-        return operationAck;
+    private String getFlowId(Map<String, String> args, String updateLocation) {
+        Optional<String> flowId = Optional.ofNullable(args)

Review Comment:
   Agreed. Removed, thanks.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r954931097


##########
minifi/minifi-integration-tests/src/test/java/org/apache/nifi/minifi/integration/c2/C2ProtocolIntegrationTest.java:
##########
@@ -0,0 +1,114 @@
+/*
+ * Licensed to the Apache Software Foundation (ASF) under one or more
+ * contributor license agreements.  See the NOTICE file distributed with
+ * this work for additional information regarding copyright ownership.
+ * The ASF licenses this file to You 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.apache.nifi.minifi.integration.c2;
+
+import com.palantir.docker.compose.DockerComposeExtension;
+import com.palantir.docker.compose.connection.waiting.HealthChecks;
+import org.apache.nifi.minifi.integration.util.LogUtil;
+import org.apache.nifi.security.util.KeystoreType;
+import org.apache.nifi.security.util.SslContextFactory;
+import org.apache.nifi.security.util.StandardTlsConfiguration;
+import org.apache.nifi.security.util.TlsConfiguration;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandalone;
+import org.apache.nifi.toolkit.tls.standalone.TlsToolkitStandaloneCommandLine;
+import org.junit.jupiter.api.AfterAll;
+import org.junit.jupiter.api.BeforeAll;
+import org.junit.jupiter.api.Test;
+import org.junit.jupiter.api.Timeout;
+
+import javax.net.ssl.SSLContext;
+import javax.net.ssl.SSLSocketFactory;
+import java.nio.file.Files;
+import java.nio.file.Path;
+import java.nio.file.Paths;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.List;
+
+@Timeout(60)
+public class C2ProtocolIntegrationTest {
+    private static final String AGENT_1 = "minifi-edge1";
+    private static final String AGENT_2 = "minifi-edge2";
+    private static final String AGENT_CLASS = "raspi3";
+    private static final String SERVICE = "c2-authoritative";
+    private static final String CONFIG_YAML = "config.text.yml.v2";
+    private static Path certificatesDirectory;
+    private static SSLContext trustSslContext;
+    private static SSLSocketFactory healthCheckSocketFactory;
+    public static DockerComposeExtension docker = DockerComposeExtension.builder()
+            .file("target/test-classes/docker-compose-c2-protocol.yml")
+            .waitingForService(AGENT_1, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .waitingForService(AGENT_2, HealthChecks.toRespond2xxOverHttp(8000, dockerPort -> "http://" + dockerPort.getIp() + ":" + dockerPort.getExternalPort()))
+            .build();
+
+    private static Path resourceDirectory;
+    private static Path authoritativeFiles;
+    private static Path minifiEdge1Version2;
+    private static Path minifiEdge2Version2;
+
+    /**
+     * Generates certificates with the tls-toolkit and then starts up the docker compose file
+     */
+    @BeforeAll
+    public static void init() throws Exception {
+        resourceDirectory = Paths.get(C2ProtocolIntegrationTest.class.getClassLoader()
+                .getResource("docker-compose-c2-protocol.yml").getFile()).getParent();
+        certificatesDirectory = resourceDirectory.toAbsolutePath().resolve("certificates-c2-protocol");
+        authoritativeFiles = resourceDirectory.resolve("c2").resolve("protocol").resolve(SERVICE).resolve("files");
+        minifiEdge1Version2 = authoritativeFiles.resolve("edge1").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+        minifiEdge2Version2 = authoritativeFiles.resolve("edge2").resolve(AGENT_CLASS).resolve(CONFIG_YAML);
+
+        if (Files.exists(minifiEdge1Version2)) {
+            Files.delete(minifiEdge1Version2);
+        }
+        if (Files.exists(minifiEdge2Version2)) {
+            Files.delete(minifiEdge2Version2);
+        }
+
+        List<String> toolkitCommandLine = new ArrayList<>(Arrays.asList("-O", "-o", certificatesDirectory.toFile().getAbsolutePath(), "-S", "badKeystorePass", "-P", "badTrustPass"));
+        for (String serverHostname : Arrays.asList(SERVICE, AGENT_1, AGENT_2)) {
+            toolkitCommandLine.add("-n");
+            toolkitCommandLine.add(serverHostname);
+        }
+        Files.createDirectories(certificatesDirectory);
+        TlsToolkitStandaloneCommandLine tlsToolkitStandaloneCommandLine = new TlsToolkitStandaloneCommandLine();
+        tlsToolkitStandaloneCommandLine.parse(toolkitCommandLine.toArray(new String[toolkitCommandLine.size()]));
+        new TlsToolkitStandalone().createNifiKeystoresAndTrustStores(tlsToolkitStandaloneCommandLine.createConfig());
+
+        TlsConfiguration tlsConfiguration = new StandardTlsConfiguration(
+                null,null,null,
+                certificatesDirectory.resolve(SERVICE).resolve("truststore.jks").toFile().getAbsolutePath(),
+                "badTrustPass",
+                KeystoreType.JKS);
+        trustSslContext = SslContextFactory.createSslContext(tlsConfiguration);
+        healthCheckSocketFactory = trustSslContext.getSocketFactory();
+
+        docker.before();
+    }
+
+    @AfterAll
+    public static void stopDocker() {
+        docker.after();
+    }
+
+    @Test
+    public void testMiNiFiHeartbeat() throws Exception {

Review Comment:
   Added the new scenario to the test. 



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r955750830


##########
c2/c2-client-bundle/c2-client-service/src/main/java/org/apache/nifi/c2/client/service/C2ClientService.java:
##########
@@ -41,8 +41,8 @@ public C2ClientService(C2Client client, C2HeartbeatFactory c2HeartbeatFactory, C
     }
 
     public void sendHeartbeat(RuntimeInfoWrapper runtimeInfoWrapper) {
-        C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);
-        client.publishHeartbeat(c2Heartbeat).ifPresent(this::processResponse);
+            C2Heartbeat c2Heartbeat = c2HeartbeatFactory.create(runtimeInfoWrapper);

Review Comment:
   Thanks, fixed it.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org


[GitHub] [nifi] rliszli commented on a diff in pull request #6281: NIFI-10312 - Fix MiNiFi C2 integration

Posted by GitBox <gi...@apache.org>.
rliszli commented on code in PR #6281:
URL: https://github.com/apache/nifi/pull/6281#discussion_r948704835


##########
c2/c2-protocol/c2-protocol-api/pom.xml:
##########
@@ -33,5 +33,9 @@ limitations under the License.
             <artifactId>c2-protocol-component-api</artifactId>
             <version>1.18.0-SNAPSHOT</version>
         </dependency>
+        <dependency>
+            <groupId>com.fasterxml.jackson.core</groupId>

Review Comment:
   The reason behind this change is that different clients sends the orderType values in different ways(upper or lower case). Before adding the @JsonCreatorit annotation, I checked and it is used several places in nifi-api modules. I can have the same result by writing a Custom Deserializer and add it to an ObjectMapper in C2JacksonSerializer. I think it's a more complicated solution and I don't think that there is any problem with adding the jackson dependency to an api module. Please let me know, if you still insist to change it, I'll do that.



-- 
This is an automated message from the Apache Git Service.
To respond to the message, please log on to GitHub and use the
URL above to go to the specific comment.

To unsubscribe, e-mail: issues-unsubscribe@nifi.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org