You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@nifi.apache.org by bb...@apache.org on 2019/08/16 20:48:38 UTC

[nifi] branch master updated (9c8f404 -> bd8342c)

This is an automated email from the ASF dual-hosted git repository.

bbende pushed a change to branch master
in repository https://gitbox.apache.org/repos/asf/nifi.git.


    from 9c8f404  NIFI-6554 fix oid parsing for PutMongo processor
     new bcf373a  NIFI-6382: Allow Parameters to have null values
     new bd8342c  NIFI-6382: On flow import and version change, create parameter context where necessary and add parameters where necessary

The 2 revisions listed above as "new" are entirely new to this
repository and will be described in separate emails.  The revisions
listed as "add" were already present in the repository and have only
been added to this reference.


Summary of changes:
 .../apache/nifi/components/ValidationContext.java  |   8 +
 .../NotificationValidationContext.java             |   5 +
 .../apache/nifi/util/MockValidationContext.java    |   8 +-
 .../org/apache/nifi/util/db/TestJdbcCommon.java    |  54 ++--
 .../nifi/controller/AbstractComponentNode.java     |  10 +
 .../apache/nifi/controller/flow/FlowManager.java   |   3 +-
 .../apache/nifi/parameter/ParameterContext.java    |  14 +-
 .../nifi/controller/StandardFlowSynchronizer.java  |   5 +-
 .../nifi/controller/flow/StandardFlowManager.java  |   2 +-
 .../serialization/StandardFlowSerializer.java      |   3 +-
 .../apache/nifi/groups/StandardProcessGroup.java   | 181 ++++++++---
 .../nifi/groups/StandardVersionedFlowStatus.java   |   4 +-
 .../nifi/parameter/StandardParameterContext.java   |  83 +++--
 .../nifi/processor/StandardValidationContext.java  |  17 +
 .../nifi/registry/flow/RestBasedFlowRegistry.java  |   2 +
 .../flow/mapping/NiFiRegistryFlowMapper.java       |  19 +-
 .../nifi/controller/TestStandardProcessorNode.java |   5 +
 .../nifi/integration/FrameworkIntegrationTest.java |  11 +-
 .../integration/MockSingleFlowRegistryClient.java  | 205 ++++++++++++
 .../nifi/integration/auth/AlwaysAuthenticate.java} |  46 +--
 .../auth/VolatileAccessPolicyProvider.java         | 118 +++++++
 .../auth/VolatileUserGroupProvider.java            | 111 +++++++
 .../nifi/integration/parameters/ParametersIT.java  |  20 +-
 .../processor/ProcessorParameterTokenIT.java       |  13 +-
 .../UsernamePasswordProcessor.java}                |  36 ++-
 .../nifi/integration/versioned/ImportFlowIT.java   | 355 ++++++++++++++++++++-
 .../parameter/TestStandardParameterContext.java    |  63 ++--
 .../resources/int-tests/clustered-nifi.properties  |   2 +-
 .../authorization/AuthorizeParameterReference.java |  50 +++
 .../org/apache/nifi/web/NiFiServiceFacade.java     |  16 +-
 .../apache/nifi/web/StandardNiFiServiceFacade.java |  37 ++-
 .../apache/nifi/web/api/ProcessGroupResource.java  |   9 +-
 .../org/apache/nifi/web/api/VersionsResource.java  |  17 +-
 .../org/apache/nifi/web/api/dto/DtoFactory.java    |   4 +-
 .../web/dao/impl/StandardParameterContextDAO.java  |  36 ++-
 .../nifi/script/impl/ValidationContextAdapter.java |   5 +
 .../stateless/core/StatelessParameterContext.java  |   8 +-
 .../stateless/core/StatelessValidationContext.java |  18 ++
 38 files changed, 1352 insertions(+), 251 deletions(-)
 create mode 100644 nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/MockSingleFlowRegistryClient.java
 copy nifi-nar-bundles/nifi-framework-bundle/nifi-framework/{nifi-web/nifi-web-api/src/test/java/org/apache/nifi/integration/util/NiFiTestLoginIdentityProvider.java => nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/AlwaysAuthenticate.java} (50%)
 create mode 100644 nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileAccessPolicyProvider.java
 create mode 100644 nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileUserGroupProvider.java
 copy nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/{cs/NopServiceReferencingProcessor.java => processors/UsernamePasswordProcessor.java} (56%)


[nifi] 01/02: NIFI-6382: Allow Parameters to have null values

Posted by bb...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

bbende pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/nifi.git

commit bcf373a049dd8d393b8cc3c36e2a26950d07a636
Author: Mark Payne <ma...@hotmail.com>
AuthorDate: Thu Aug 8 10:54:03 2019 -0400

    NIFI-6382: Allow Parameters to have null values
    
    Signed-off-by: Bryan Bende <bb...@apache.org>
---
 .../apache/nifi/components/ValidationContext.java  |   8 ++
 .../NotificationValidationContext.java             |   5 +
 .../apache/nifi/util/MockValidationContext.java    |   8 +-
 .../org/apache/nifi/util/db/TestJdbcCommon.java    |  54 +++++-----
 .../nifi/controller/AbstractComponentNode.java     |   8 ++
 .../apache/nifi/controller/flow/FlowManager.java   |   3 +-
 .../apache/nifi/parameter/ParameterContext.java    |  14 +--
 .../nifi/controller/StandardFlowSynchronizer.java  |   5 +-
 .../nifi/controller/flow/StandardFlowManager.java  |   2 +-
 .../nifi/parameter/StandardParameterContext.java   |  83 +++++++--------
 .../nifi/processor/StandardValidationContext.java  |  17 +++
 .../nifi/controller/TestStandardProcessorNode.java |   5 +
 .../nifi/integration/auth/AlwaysAuthenticate.java  |  46 ++++++++
 .../auth/VolatileAccessPolicyProvider.java         | 118 +++++++++++++++++++++
 .../auth/VolatileUserGroupProvider.java            | 111 +++++++++++++++++++
 .../nifi/integration/parameters/ParametersIT.java  |  20 ++--
 .../processor/ProcessorParameterTokenIT.java       |  13 ++-
 .../nifi/integration/versioned/ImportFlowIT.java   |  22 +++-
 .../parameter/TestStandardParameterContext.java    |  63 ++++++-----
 .../resources/int-tests/clustered-nifi.properties  |   2 +-
 .../apache/nifi/web/StandardNiFiServiceFacade.java |   9 +-
 .../web/dao/impl/StandardParameterContextDAO.java  |  36 +++++--
 .../nifi/script/impl/ValidationContextAdapter.java |   5 +
 .../stateless/core/StatelessParameterContext.java  |   8 +-
 .../stateless/core/StatelessValidationContext.java |  18 ++++
 25 files changed, 530 insertions(+), 153 deletions(-)

diff --git a/nifi-api/src/main/java/org/apache/nifi/components/ValidationContext.java b/nifi-api/src/main/java/org/apache/nifi/components/ValidationContext.java
index 95a8c09..acaffd7 100644
--- a/nifi-api/src/main/java/org/apache/nifi/components/ValidationContext.java
+++ b/nifi-api/src/main/java/org/apache/nifi/components/ValidationContext.java
@@ -106,4 +106,12 @@ public interface ValidationContext extends PropertyContext {
      * @return <code>true</code> if a Parameter with the given name is defined in the currently selected Parameter Context
      */
     boolean isParameterDefined(String parameterName);
+
+    /**
+     * Returns <code>true</code> if a Parameter with the given name is defined and has a non-null value, <code>false</code> if either the Parameter
+     * is not defined or the Parameter is defined but has a value of <code>null</code>.
+     * @param parameterName the name of the parameter
+     * @return <code>true</code> if the Parameter is defined and has a non-null value, false otherwise
+     */
+    boolean isParameterSet(String parameterName);
 }
diff --git a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/notification/NotificationValidationContext.java b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/notification/NotificationValidationContext.java
index 8f7eda0..0075df2 100644
--- a/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/notification/NotificationValidationContext.java
+++ b/nifi-bootstrap/src/main/java/org/apache/nifi/bootstrap/notification/NotificationValidationContext.java
@@ -133,4 +133,9 @@ public class NotificationValidationContext implements ValidationContext {
     public boolean isParameterDefined(final String parameterName) {
         return false;
     }
+
+    @Override
+    public boolean isParameterSet(final String parameterName) {
+        return false;
+    }
 }
diff --git a/nifi-mock/src/main/java/org/apache/nifi/util/MockValidationContext.java b/nifi-mock/src/main/java/org/apache/nifi/util/MockValidationContext.java
index 87389be..e913204 100644
--- a/nifi-mock/src/main/java/org/apache/nifi/util/MockValidationContext.java
+++ b/nifi-mock/src/main/java/org/apache/nifi/util/MockValidationContext.java
@@ -181,8 +181,12 @@ public class MockValidationContext extends MockControllerServiceLookup implement
 
     @Override
     public boolean isParameterDefined(final String parameterName) {
-        // TODO: Implement
-        return false;
+        return true;
+    }
+
+    @Override
+    public boolean isParameterSet(final String parameterName) {
+        return true;
     }
 
 }
diff --git a/nifi-nar-bundles/nifi-extension-utils/nifi-database-utils/src/test/java/org/apache/nifi/util/db/TestJdbcCommon.java b/nifi-nar-bundles/nifi-extension-utils/nifi-database-utils/src/test/java/org/apache/nifi/util/db/TestJdbcCommon.java
index b58289b..bbf2bdb 100644
--- a/nifi-nar-bundles/nifi-extension-utils/nifi-database-utils/src/test/java/org/apache/nifi/util/db/TestJdbcCommon.java
+++ b/nifi-nar-bundles/nifi-extension-utils/nifi-database-utils/src/test/java/org/apache/nifi/util/db/TestJdbcCommon.java
@@ -16,13 +16,26 @@
  */
 package org.apache.nifi.util.db;
 
-import static org.junit.Assert.assertEquals;
-import static org.junit.Assert.assertNotNull;
-import static org.junit.Assert.assertNull;
-import static org.junit.Assert.assertTrue;
-import static org.mockito.ArgumentMatchers.anyInt;
-import static org.mockito.Mockito.mock;
-import static org.mockito.Mockito.when;
+import org.apache.avro.Conversions;
+import org.apache.avro.LogicalType;
+import org.apache.avro.LogicalTypes;
+import org.apache.avro.Schema;
+import org.apache.avro.file.DataFileStream;
+import org.apache.avro.generic.GenericData;
+import org.apache.avro.generic.GenericDatumReader;
+import org.apache.avro.generic.GenericRecord;
+import org.apache.avro.io.DatumReader;
+import org.apache.avro.util.Utf8;
+import org.apache.commons.io.input.ReaderInputStream;
+import org.junit.Assert;
+import org.junit.BeforeClass;
+import org.junit.ClassRule;
+import org.junit.Test;
+import org.junit.rules.TemporaryFolder;
+import org.mockito.Mockito;
+import org.mockito.stubbing.Answer;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
 
 import java.io.ByteArrayInputStream;
 import java.io.ByteArrayOutputStream;
@@ -63,26 +76,13 @@ import java.util.function.BiConsumer;
 import java.util.function.BiFunction;
 import java.util.stream.IntStream;
 
-import org.apache.avro.Conversions;
-import org.apache.avro.LogicalType;
-import org.apache.avro.LogicalTypes;
-import org.apache.avro.Schema;
-import org.apache.avro.file.DataFileStream;
-import org.apache.avro.generic.GenericData;
-import org.apache.avro.generic.GenericDatumReader;
-import org.apache.avro.generic.GenericRecord;
-import org.apache.avro.io.DatumReader;
-import org.apache.avro.util.Utf8;
-import org.apache.commons.io.input.ReaderInputStream;
-import org.junit.Assert;
-import org.junit.BeforeClass;
-import org.junit.ClassRule;
-import org.junit.Test;
-import org.junit.rules.TemporaryFolder;
-import org.mockito.Mockito;
-import org.mockito.stubbing.Answer;
-import org.slf4j.Logger;
-import org.slf4j.LoggerFactory;
+import static org.junit.Assert.assertEquals;
+import static org.junit.Assert.assertNotNull;
+import static org.junit.Assert.assertNull;
+import static org.junit.Assert.assertTrue;
+import static org.mockito.ArgumentMatchers.anyInt;
+import static org.mockito.Mockito.mock;
+import static org.mockito.Mockito.when;
 
 public class TestJdbcCommon {
 
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java
index f5752a3..48e359f 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java
@@ -659,6 +659,14 @@ public abstract class AbstractComponentNode implements ComponentNode {
                         .explanation("Property references Parameter '" + paramName + "' but the currently selected Parameter Context does not have a Parameter with that name")
                         .build());
                 }
+
+                if (!validationContext.isParameterSet(paramName)) {
+                    results.add(new ValidationResult.Builder()
+                        .subject(propertyDescriptor.getDisplayName())
+                        .valid(false)
+                        .explanation("Property references Parameter '" + paramName + "' but the currently selected Parameter Context does not have a value set for that Parameter")
+                        .build());
+                }
             }
         }
 
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/flow/FlowManager.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/flow/FlowManager.java
index 4d108e9..d072698 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/flow/FlowManager.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/flow/FlowManager.java
@@ -36,6 +36,7 @@ import org.apache.nifi.web.api.dto.FlowSnippetDTO;
 
 import java.net.URL;
 import java.util.Collection;
+import java.util.Map;
 import java.util.Optional;
 import java.util.Set;
 
@@ -322,7 +323,7 @@ public interface FlowManager {
 
     void removeRootControllerService(final ControllerServiceNode service);
 
-    ParameterContext createParameterContext(String id, String name, Set<Parameter> parameters);
+    ParameterContext createParameterContext(String id, String name, Map<String, Parameter> parameters);
 
     ParameterContextManager getParameterContextManager();
 }
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/parameter/ParameterContext.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/parameter/ParameterContext.java
index dae4d9e..fcbd270 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/parameter/ParameterContext.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/parameter/ParameterContext.java
@@ -20,7 +20,6 @@ import org.apache.nifi.authorization.resource.Authorizable;
 
 import java.util.Map;
 import java.util.Optional;
-import java.util.Set;
 
 public interface ParameterContext extends ParameterLookup, Authorizable {
 
@@ -52,19 +51,22 @@ public interface ParameterContext extends ParameterLookup, Authorizable {
     void setDescription(String description);
 
     /**
-     * Updates the Parameters within this context to match the given set of Parameters.
-     * @param updatedParameters the updated set of parameters
+     * Updates the Parameters within this context to match the given set of Parameters. If the Parameter Context contains any parameters that are not in
+     * the given set of updated Parameters, those parameters are unaffected. However, if the Map contains any key with a <code>null</code> value, the
+     * parameter whose name is given by the key will be removed
+     *
+     * @param updatedParameters the updated set of parameters, keyed by Parameter name
      * @throws IllegalStateException if any parameter is modified or removed and that parameter is being referenced by a running Processor or an enabled Controller Service, or if
      * an update would result in changing the sensitivity of any parameter
      */
-    void setParameters(Set<Parameter> updatedParameters);
+    void setParameters(Map<String, Parameter> updatedParameters);
 
     /**
      * Ensures that it is legal to update the Parameters for this Parameter Context to match the given set of Parameters
-     * @param parameters the Set of Parameters that are to become the new Parameters for this Parameter Context
+     * @param parameters the updated set of parameters, keyed by Parameter name
      * @throws IllegalStateException if setting the given set of Parameters is not legal
      */
-    void verifyCanSetParameters(Set<Parameter> parameters);
+    void verifyCanSetParameters(Map<String, Parameter> parameters);
 
 
     /**
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/StandardFlowSynchronizer.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/StandardFlowSynchronizer.java
index a2c8d4c..6d970db 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/StandardFlowSynchronizer.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/StandardFlowSynchronizer.java
@@ -131,6 +131,7 @@ import java.util.Map;
 import java.util.Objects;
 import java.util.Set;
 import java.util.concurrent.TimeUnit;
+import java.util.function.Function;
 import java.util.stream.Collectors;
 import java.util.zip.GZIPInputStream;
 
@@ -547,10 +548,10 @@ public class StandardFlowSynchronizer implements FlowSynchronizer {
     }
 
     private ParameterContext createParameterContext(final ParameterContextDTO dto, final FlowManager flowManager) {
-        final Set<Parameter> parameters = dto.getParameters().stream()
+        final Map<String, Parameter> parameters = dto.getParameters().stream()
             .map(ParameterEntity::getParameter)
             .map(this::createParameter)
-            .collect(Collectors.toSet());
+            .collect(Collectors.toMap(param -> param.getDescriptor().getName(), Function.identity()));
 
         final ParameterContext context = flowManager.createParameterContext(dto.getId(), dto.getName(), parameters);
         context.setDescription(dto.getDescription());
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/flow/StandardFlowManager.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/flow/StandardFlowManager.java
index 1ec226e..f2f1bc7 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/flow/StandardFlowManager.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/flow/StandardFlowManager.java
@@ -736,7 +736,7 @@ public class StandardFlowManager implements FlowManager {
     }
 
     @Override
-    public ParameterContext createParameterContext(final String id, final String name, final Set<Parameter> parameters) {
+    public ParameterContext createParameterContext(final String id, final String name, final Map<String, Parameter> parameters) {
         final boolean namingConflict = parameterContextManager.getParameterContexts().stream()
             .anyMatch(paramContext -> paramContext.getName().equals(name));
 
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/parameter/StandardParameterContext.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/parameter/StandardParameterContext.java
index e5d12a5..04cfb8a 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/parameter/StandardParameterContext.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/parameter/StandardParameterContext.java
@@ -34,7 +34,6 @@ import java.util.LinkedHashMap;
 import java.util.Map;
 import java.util.Objects;
 import java.util.Optional;
-import java.util.Set;
 import java.util.concurrent.locks.Lock;
 import java.util.concurrent.locks.ReadWriteLock;
 import java.util.concurrent.locks.ReentrantReadWriteLock;
@@ -97,32 +96,26 @@ public class StandardParameterContext implements ParameterContext {
         return description;
     }
 
-    public void setParameters(final Set<Parameter> updatedParameters) {
+    public void setParameters(final Map<String, Parameter> updatedParameters) {
         writeLock.lock();
         try {
             this.version++;
-
             verifyCanSetParameters(updatedParameters);
 
             boolean changeAffectingComponents = false;
-            for (final Parameter parameter : updatedParameters) {
-                if (parameter.getValue() == null && parameter.getDescriptor().getDescription() == null) {
-                    parameters.remove(parameter.getDescriptor());
+            for (final Map.Entry<String, Parameter> entry : updatedParameters.entrySet()) {
+                final String parameterName = entry.getKey();
+                final Parameter parameter = entry.getValue();
+
+                if (parameter == null) {
+                    final ParameterDescriptor parameterDescriptor = new ParameterDescriptor.Builder().name(parameterName).build();
+                    parameters.remove(parameterDescriptor);
                     changeAffectingComponents = true;
-                } else if (parameter.getValue() == null) {
-                    // Value is null but description is not. Just update the description of the existing Parameter.
-                    final Parameter existingParameter = parameters.get(parameter.getDescriptor());
-                    final ParameterDescriptor existingDescriptor = existingParameter.getDescriptor();
-                    final ParameterDescriptor replacementDescriptor = new ParameterDescriptor.Builder()
-                        .from(existingDescriptor)
-                        .description(parameter.getDescriptor().getDescription())
-                        .build();
-
-                    final Parameter replacementParameter = new Parameter(replacementDescriptor, existingParameter.getValue());
-                    parameters.put(parameter.getDescriptor(), replacementParameter);
                 } else {
-                    parameters.put(parameter.getDescriptor(), parameter);
-                    changeAffectingComponents = true;
+                    final Parameter oldParameter = parameters.put(parameter.getDescriptor(), parameter);
+                    if (oldParameter == null || !Objects.equals(oldParameter.getValue(), parameter.getValue())) {
+                        changeAffectingComponents = true;
+                    }
                 }
             }
 
@@ -195,25 +188,23 @@ public class StandardParameterContext implements ParameterContext {
     }
 
     @Override
-    public void verifyCanSetParameters(final Set<Parameter> updatedParameters) {
+    public void verifyCanSetParameters(final Map<String, Parameter> updatedParameters) {
         // Ensure that the updated parameters will not result in changing the sensitivity flag of any parameter.
-        for (final Parameter updatedParameter : updatedParameters) {
-            validateSensitiveFlag(updatedParameter);
-
-            // Parameters' names and sensitivity flags are immutable. However, the description and value are mutable. If both value and description are
-            // set to `null`, this is the indication that the Parameter should be removed. If the value is `null` but the Description is supplied, the user
-            // is indicating that only the description is to be changed.
-            if (updatedParameter.getValue() == null && updatedParameter.getDescriptor().getDescription() == null) {
-                validateReferencingComponents(updatedParameter, "remove");
-            } else if (updatedParameter.getValue() != null) {
-                validateReferencingComponents(updatedParameter, "update");
-            } else {
-                // Only parameter is changing. No value is set. This means that the Parameter must already exist.
-                final Optional<Parameter> existing = getParameter(updatedParameter.getDescriptor());
-                if (!existing.isPresent()) {
-                    throw new IllegalStateException("Cannot add Parameter '" + updatedParameter.getDescriptor().getName() + "' without providing a value");
-                }
+        for (final Map.Entry<String, Parameter> entry : updatedParameters.entrySet()) {
+            final String parameterName = entry.getKey();
+            final Parameter parameter = entry.getValue();
+            if (parameter == null) {
+                // parameter is being deleted.
+                validateReferencingComponents(parameterName, null,"remove");
+                continue;
             }
+
+            if (!Objects.equals(parameterName, parameter.getDescriptor().getName())) {
+                throw new IllegalArgumentException("Parameter '" + parameterName + "' was specified with the wrong key in the Map");
+            }
+
+            validateSensitiveFlag(parameter);
+            validateReferencingComponents(parameterName, parameter, "update");
         }
     }
 
@@ -236,25 +227,27 @@ public class StandardParameterContext implements ParameterContext {
     }
 
 
-    private void validateReferencingComponents(final Parameter updatedParameter, final String parameterAction) {
-        final String paramName = updatedParameter.getDescriptor().getName();
-
-        for (final ProcessorNode procNode : parameterReferenceManager.getProcessorsReferencing(this, paramName)) {
+    private void validateReferencingComponents(final String parameterName, final Parameter parameter, final String parameterAction) {
+        for (final ProcessorNode procNode : parameterReferenceManager.getProcessorsReferencing(this, parameterName)) {
             if (procNode.isRunning()) {
-                throw new IllegalStateException("Cannot " + parameterAction + " parameter '" + paramName + "' because it is referenced by " + procNode + ", which is currently running");
+                throw new IllegalStateException("Cannot " + parameterAction + " parameter '" + parameterName + "' because it is referenced by " + procNode + ", which is currently running");
             }
 
-            validateParameterSensitivity(updatedParameter, procNode);
+            if (parameter != null) {
+                validateParameterSensitivity(parameter, procNode);
+            }
         }
 
-        for (final ControllerServiceNode serviceNode : parameterReferenceManager.getControllerServicesReferencing(this, paramName)) {
+        for (final ControllerServiceNode serviceNode : parameterReferenceManager.getControllerServicesReferencing(this, parameterName)) {
             final ControllerServiceState serviceState = serviceNode.getState();
             if (serviceState != ControllerServiceState.DISABLED) {
-                throw new IllegalStateException("Cannot " + parameterAction + " parameter '" + paramName + "' because it is referenced by "
+                throw new IllegalStateException("Cannot " + parameterAction + " parameter '" + parameterName + "' because it is referenced by "
                     + serviceNode + ", which currently has a state of " + serviceState);
             }
 
-            validateParameterSensitivity(updatedParameter, serviceNode);
+            if (parameter != null) {
+                validateParameterSensitivity(parameter, serviceNode);
+            }
         }
     }
 
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/processor/StandardValidationContext.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/processor/StandardValidationContext.java
index b1a44e7..3fc7e06 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/processor/StandardValidationContext.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/processor/StandardValidationContext.java
@@ -33,6 +33,7 @@ import org.apache.nifi.controller.service.ControllerServiceProvider;
 import org.apache.nifi.controller.service.ControllerServiceState;
 import org.apache.nifi.expression.ExpressionLanguageCompiler;
 import org.apache.nifi.groups.ProcessGroup;
+import org.apache.nifi.parameter.Parameter;
 import org.apache.nifi.parameter.ParameterContext;
 import org.apache.nifi.parameter.ParameterReference;
 import org.apache.nifi.registry.VariableRegistry;
@@ -43,6 +44,7 @@ import java.util.HashMap;
 import java.util.LinkedHashMap;
 import java.util.List;
 import java.util.Map;
+import java.util.Optional;
 import java.util.concurrent.atomic.AtomicReference;
 import java.util.stream.Collectors;
 
@@ -217,6 +219,21 @@ public class StandardValidationContext implements ValidationContext {
     }
 
     @Override
+    public boolean isParameterSet(final String parameterName) {
+        if (parameterContext == null) {
+            return false;
+        }
+
+        final Optional<Parameter> parameterOption = parameterContext.getParameter(parameterName);
+        if (!parameterOption.isPresent()) {
+            return false;
+        }
+
+        final String value = parameterOption.get().getValue();
+        return value != null;
+    }
+
+    @Override
     public String toString() {
         return "StandardValidationContext[componentId=" + componentId + ", properties=" + properties + "]";
     }
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/controller/TestStandardProcessorNode.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/controller/TestStandardProcessorNode.java
index 08dc4f3..d1fb511 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/controller/TestStandardProcessorNode.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/controller/TestStandardProcessorNode.java
@@ -550,6 +550,11 @@ public class TestStandardProcessorNode {
                     public boolean isParameterDefined(final String parameterName) {
                         return false;
                     }
+
+                    @Override
+                    public boolean isParameterSet(final String parameterName) {
+                        return false;
+                    }
                 };
             }
 
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/AlwaysAuthenticate.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/AlwaysAuthenticate.java
new file mode 100644
index 0000000..9640be5
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/AlwaysAuthenticate.java
@@ -0,0 +1,46 @@
+/*
+ * 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.integration.auth;
+
+import org.apache.nifi.authentication.AuthenticationResponse;
+import org.apache.nifi.authentication.LoginCredentials;
+import org.apache.nifi.authentication.LoginIdentityProvider;
+import org.apache.nifi.authentication.LoginIdentityProviderConfigurationContext;
+import org.apache.nifi.authentication.LoginIdentityProviderInitializationContext;
+import org.apache.nifi.authentication.exception.IdentityAccessException;
+import org.apache.nifi.authentication.exception.InvalidLoginCredentialsException;
+import org.apache.nifi.authentication.exception.ProviderCreationException;
+import org.apache.nifi.authentication.exception.ProviderDestructionException;
+
+public class AlwaysAuthenticate implements LoginIdentityProvider {
+    @Override
+    public AuthenticationResponse authenticate(final LoginCredentials credentials) throws InvalidLoginCredentialsException, IdentityAccessException {
+        return new AuthenticationResponse(credentials.getUsername(), credentials.getUsername(), 1_000_000L, "unit-test-issuer");
+    }
+
+    @Override
+    public void initialize(final LoginIdentityProviderInitializationContext initializationContext) throws ProviderCreationException {
+    }
+
+    @Override
+    public void onConfigured(final LoginIdentityProviderConfigurationContext configurationContext) throws ProviderCreationException {
+    }
+
+    @Override
+    public void preDestruction() throws ProviderDestructionException {
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileAccessPolicyProvider.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileAccessPolicyProvider.java
new file mode 100644
index 0000000..c010f05
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileAccessPolicyProvider.java
@@ -0,0 +1,118 @@
+/*
+ * 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.integration.auth;
+
+import org.apache.nifi.authorization.AccessPolicy;
+import org.apache.nifi.authorization.AccessPolicyProvider;
+import org.apache.nifi.authorization.AccessPolicyProviderInitializationContext;
+import org.apache.nifi.authorization.AuthorizerConfigurationContext;
+import org.apache.nifi.authorization.RequestAction;
+import org.apache.nifi.authorization.exception.AuthorizationAccessException;
+import org.apache.nifi.authorization.exception.AuthorizerCreationException;
+import org.apache.nifi.authorization.exception.AuthorizerDestructionException;
+
+import java.util.HashSet;
+import java.util.Objects;
+import java.util.Set;
+
+public class VolatileAccessPolicyProvider implements AccessPolicyProvider {
+    private final VolatileUserGroupProvider userGroupProvider = new VolatileUserGroupProvider();
+    private Set<AccessPolicy> accessPolicies = new HashSet<>();
+
+    public synchronized void grantAccess(final String user, final String resourceIdentifier, final RequestAction action) {
+        final AccessPolicy existingPolicy = getAccessPolicy(resourceIdentifier, action);
+
+        final AccessPolicy policy;
+        if (existingPolicy == null) {
+            policy = new AccessPolicy.Builder()
+                .addUser(user)
+                .action(action)
+                .identifierGenerateRandom()
+                .resource(resourceIdentifier)
+                .build();
+        } else {
+            policy = new AccessPolicy.Builder()
+                .addUsers(existingPolicy.getUsers())
+                .addUser(user)
+                .action(action)
+                .identifier(existingPolicy.getIdentifier())
+                .resource(resourceIdentifier)
+                .build();
+        }
+
+        accessPolicies.remove(existingPolicy);
+        accessPolicies.add(policy);
+    }
+
+    public synchronized void revokeAccess(final String user, final String resourceIdentifier, final RequestAction action) {
+        final AccessPolicy existingPolicy = getAccessPolicy(resourceIdentifier, action);
+
+        if (existingPolicy == null) {
+            return;
+        }
+
+        final AccessPolicy policy= new AccessPolicy.Builder()
+                .addUsers(existingPolicy.getUsers())
+                .removeUser(user)
+                .action(action)
+                .identifier(existingPolicy.getIdentifier())
+                .resource(resourceIdentifier)
+                .build();
+
+        accessPolicies.remove(existingPolicy);
+        accessPolicies.add(policy);
+    }
+
+    @Override
+    public synchronized Set<AccessPolicy> getAccessPolicies() throws AuthorizationAccessException {
+        return new HashSet<>(accessPolicies);
+    }
+
+    @Override
+    public synchronized AccessPolicy getAccessPolicy(final String identifier) throws AuthorizationAccessException {
+        return accessPolicies.stream()
+            .filter(policy -> policy.getIdentifier().equals(identifier))
+            .findAny()
+            .orElse(null);
+    }
+
+    @Override
+    public synchronized AccessPolicy getAccessPolicy(final String resourceIdentifier, final RequestAction action) throws AuthorizationAccessException {
+        return accessPolicies.stream()
+            .filter(policy -> Objects.equals(policy.getResource(), resourceIdentifier))
+            .filter(policy -> Objects.equals(policy.getAction(), action))
+            .findAny()
+            .orElse(null);
+    }
+
+    @Override
+    public synchronized VolatileUserGroupProvider getUserGroupProvider() {
+        return userGroupProvider;
+    }
+
+    @Override
+    public void initialize(final AccessPolicyProviderInitializationContext initializationContext) throws AuthorizerCreationException {
+    }
+
+    @Override
+    public void onConfigured(final AuthorizerConfigurationContext configurationContext) throws AuthorizerCreationException {
+    }
+
+    @Override
+    public void preDestruction() throws AuthorizerDestructionException {
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileUserGroupProvider.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileUserGroupProvider.java
new file mode 100644
index 0000000..7e4849b
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/auth/VolatileUserGroupProvider.java
@@ -0,0 +1,111 @@
+/*
+ * 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.integration.auth;
+
+import org.apache.nifi.authorization.AuthorizerConfigurationContext;
+import org.apache.nifi.authorization.Group;
+import org.apache.nifi.authorization.User;
+import org.apache.nifi.authorization.UserAndGroups;
+import org.apache.nifi.authorization.UserGroupProvider;
+import org.apache.nifi.authorization.UserGroupProviderInitializationContext;
+import org.apache.nifi.authorization.exception.AuthorizationAccessException;
+import org.apache.nifi.authorization.exception.AuthorizerCreationException;
+import org.apache.nifi.authorization.exception.AuthorizerDestructionException;
+
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.Map;
+import java.util.Objects;
+import java.util.Set;
+
+public class VolatileUserGroupProvider implements UserGroupProvider {
+    private final Map<String, User> users = new HashMap<>();
+    private final Map<String, Group> groups = new HashMap<>();
+    private final Map<User, Set<Group>> userGroupMapping = new HashMap<>();
+
+    public void addUser(final User user) {
+        this.users.put(user.getIdentifier(), user);
+    }
+
+    public void addGroup(final Group group) {
+        this.groups.put(group.getIdentifier(), group);
+    }
+
+    public void addUserToGroup(final User user, final Group group) {
+        userGroupMapping.computeIfAbsent(user, i -> new HashSet<>()).add(group);
+    }
+
+    @Override
+    public Set<User> getUsers() throws AuthorizationAccessException {
+        return new HashSet<>(users.values());
+    }
+
+    @Override
+    public User getUser(final String identifier) throws AuthorizationAccessException {
+        return users.get(identifier);
+    }
+
+    @Override
+    public User getUserByIdentity(final String identity) throws AuthorizationAccessException {
+        return users.values().stream()
+            .filter(user -> Objects.equals(identity, user.getIdentity()))
+            .findFirst()
+            .orElse(null);
+    }
+
+    @Override
+    public Set<Group> getGroups() throws AuthorizationAccessException {
+        return new HashSet<>(groups.values());
+    }
+
+    @Override
+    public Group getGroup(final String identifier) throws AuthorizationAccessException {
+        return groups.get(identifier);
+    }
+
+    @Override
+    public UserAndGroups getUserAndGroups(final String identity) throws AuthorizationAccessException {
+        final User user = getUserByIdentity(identity);
+        final Set<Group> groups = userGroupMapping.get(user);
+        final Set<Group> groupCopy = groups == null ? Collections.emptySet() : new HashSet<>(groups);
+
+        return new UserAndGroups() {
+            @Override
+            public User getUser() {
+                return user;
+            }
+
+            @Override
+            public Set<Group> getGroups() {
+                return groupCopy;
+            }
+        };
+    }
+
+    @Override
+    public void initialize(final UserGroupProviderInitializationContext initializationContext) throws AuthorizerCreationException {
+    }
+
+    @Override
+    public void onConfigured(final AuthorizerConfigurationContext configurationContext) throws AuthorizerCreationException {
+    }
+
+    @Override
+    public void preDestruction() throws AuthorizerDestructionException {
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/parameters/ParametersIT.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/parameters/ParametersIT.java
index 7d0755d..4a394ae 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/parameters/ParametersIT.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/parameters/ParametersIT.java
@@ -58,7 +58,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
         updateAttribute.setProperties(Collections.singletonMap("test", "#{test}"));
@@ -83,7 +83,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
         updateAttribute.setProperties(Collections.singletonMap("test", "${#{test}:toUpper()}"));
@@ -108,7 +108,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
         updateAttribute.setProperties(Collections.singletonMap("test", "${#{test}:toUpper()}"));
@@ -133,7 +133,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
 
@@ -165,7 +165,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
 
@@ -193,7 +193,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
 
@@ -225,7 +225,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
 
@@ -267,7 +267,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
 
@@ -307,7 +307,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
 
@@ -337,7 +337,7 @@ public class ParametersIT extends FrameworkIntegrationTest {
 
         final ParameterReferenceManager referenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "param-context", referenceManager, null);
-        parameterContext.setParameters(Collections.singleton(new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
+        parameterContext.setParameters(Collections.singletonMap("test", new Parameter(new ParameterDescriptor.Builder().name("test").build(), "unit")));
 
         getRootGroup().setParameterContext(parameterContext);
 
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/processor/ProcessorParameterTokenIT.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/processor/ProcessorParameterTokenIT.java
index 3843d2a..906b092 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/processor/ProcessorParameterTokenIT.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/processor/ProcessorParameterTokenIT.java
@@ -40,7 +40,6 @@ import java.nio.charset.StandardCharsets;
 import java.util.Arrays;
 import java.util.Collections;
 import java.util.HashMap;
-import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
 import java.util.Set;
@@ -77,9 +76,9 @@ public class ProcessorParameterTokenIT extends FrameworkIntegrationTest {
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "testEscapedParameterReference", ParameterReferenceManager.EMPTY, null);
         getRootGroup().setParameterContext(parameterContext);
 
-        final Set<Parameter> parameters = new HashSet<>();
-        parameters.add(new Parameter(new ParameterDescriptor.Builder().name("foo").build(), "bar"));
-        parameters.add(new Parameter(new ParameterDescriptor.Builder().name("sensitive").sensitive(true).build(), "*password*"));
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameters.put("foo", new Parameter(new ParameterDescriptor.Builder().name("foo").build(), "bar"));
+        parameters.put("sensitive", new Parameter(new ParameterDescriptor.Builder().name("sensitive").sensitive(true).build(), "*password*"));
         parameterContext.setParameters(parameters);
 
         verifyText(procNode, "hello", "hello");
@@ -107,9 +106,9 @@ public class ProcessorParameterTokenIT extends FrameworkIntegrationTest {
         final ParameterContext parameterContext = new StandardParameterContext(UUID.randomUUID().toString(), "testEscapedParameterReference", ParameterReferenceManager.EMPTY, null);
         getRootGroup().setParameterContext(parameterContext);
 
-        final Set<Parameter> parameters = new HashSet<>();
-        parameters.add(new Parameter(new ParameterDescriptor.Builder().name("foo").build(), "bar"));
-        parameters.add(new Parameter(new ParameterDescriptor.Builder().name("sensitive").sensitive(true).build(), "*password*"));
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameters.put("foo", new Parameter(new ParameterDescriptor.Builder().name("foo").build(), "bar"));
+        parameters.put("sensitive", new Parameter(new ParameterDescriptor.Builder().name("sensitive").sensitive(true).build(), "*password*"));
         parameterContext.setParameters(parameters);
 
         verifyCannotSetParameter(procNode, "#{sensitive}", null);
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java
index 5206315..99f91b8 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java
@@ -34,6 +34,8 @@ import org.apache.nifi.registry.flow.VersionedControllerService;
 import org.apache.nifi.registry.flow.VersionedFlow;
 import org.apache.nifi.registry.flow.VersionedFlowSnapshot;
 import org.apache.nifi.registry.flow.VersionedFlowSnapshotMetadata;
+import org.apache.nifi.registry.flow.VersionedParameter;
+import org.apache.nifi.registry.flow.VersionedParameterContext;
 import org.apache.nifi.registry.flow.VersionedProcessGroup;
 import org.apache.nifi.registry.flow.VersionedProcessor;
 import org.apache.nifi.registry.flow.mapping.NiFiRegistryFlowMapper;
@@ -44,6 +46,7 @@ import java.util.Collections;
 import java.util.HashMap;
 import java.util.HashSet;
 import java.util.List;
+import java.util.Map;
 import java.util.Set;
 import java.util.concurrent.ExecutionException;
 
@@ -70,7 +73,7 @@ public class ImportFlowIT extends FrameworkIntegrationTest {
         processor.setAutoTerminatedRelationships(Collections.singleton(REL_SUCCESS));
         processor.setProperties(Collections.singletonMap(NopServiceReferencingProcessor.SERVICE.getName(), controllerService.getIdentifier()));
 
-        final VersionedFlowSnapshot proposedFlow = createFlowSnapshot(Collections.singletonList(controllerService), Collections.singletonList(processor));
+        final VersionedFlowSnapshot proposedFlow = createFlowSnapshot(Collections.singletonList(controllerService), Collections.singletonList(processor), null);
 
         // Create an Inner Process Group and update it to match the Versioned Flow.
         final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
@@ -107,7 +110,7 @@ public class ImportFlowIT extends FrameworkIntegrationTest {
     }
 
 
-    private VersionedFlowSnapshot createFlowSnapshot(final List<ControllerServiceNode> controllerServices, final List<ProcessorNode> processors) {
+    private VersionedFlowSnapshot createFlowSnapshot(final List<ControllerServiceNode> controllerServices, final List<ProcessorNode> processors, final Map<String, String> parameters) {
         final VersionedFlowSnapshotMetadata snapshotMetadata = new VersionedFlowSnapshotMetadata();
         snapshotMetadata.setAuthor("unit-test");
         snapshotMetadata.setBucketIdentifier("unit-test-bucket");
@@ -159,6 +162,21 @@ public class ImportFlowIT extends FrameworkIntegrationTest {
         versionedFlowSnapshot.setFlow(flow);
         versionedFlowSnapshot.setFlowContents(flowContents);
 
+        if (parameters != null) {
+            final Set<VersionedParameter> versionedParameters = new HashSet<>();
+            for (final Map.Entry<String, String> entry : parameters.entrySet()) {
+                final VersionedParameter versionedParameter = new VersionedParameter();
+                versionedParameter.setName(entry.getKey());
+                versionedParameter.setValue(entry.getValue());
+                versionedParameters.add(versionedParameter);
+            }
+
+            final VersionedParameterContext versionedParameterContext = new VersionedParameterContext();
+            versionedParameterContext.setName("Unit Test Context");
+            versionedParameterContext.setParameters(versionedParameters);
+            versionedFlowSnapshot.setParameterContexts(Collections.singletonMap("unit-test-context", versionedParameterContext));
+        }
+
         return versionedFlowSnapshot;
     }
 }
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/parameter/TestStandardParameterContext.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/parameter/TestStandardParameterContext.java
index 1b211a2..6008e63 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/parameter/TestStandardParameterContext.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/parameter/TestStandardParameterContext.java
@@ -27,7 +27,6 @@ import org.mockito.Mockito;
 import java.util.Collections;
 import java.util.EnumSet;
 import java.util.HashMap;
-import java.util.HashSet;
 import java.util.Map;
 import java.util.Set;
 
@@ -47,9 +46,9 @@ public class TestStandardParameterContext {
         final ParameterDescriptor xyzDescriptor = new ParameterDescriptor.Builder().name("xyz").build();
         final ParameterDescriptor fooDescriptor = new ParameterDescriptor.Builder().name("foo").description("bar").sensitive(true).build();
 
-        final Set<Parameter> parameters = new HashSet<>();
-        parameters.add(new Parameter(abcDescriptor, "123"));
-        parameters.add(new Parameter(xyzDescriptor, "242526"));
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameters.put("abc", new Parameter(abcDescriptor, "123"));
+        parameters.put("xyz", new Parameter(xyzDescriptor, "242526"));
 
         context.setParameters(parameters);
 
@@ -63,15 +62,15 @@ public class TestStandardParameterContext {
         assertNull(xyzParam.getDescriptor().getDescription());
         assertEquals("242526", xyzParam.getValue());
 
-        final Set<Parameter> secondParameters = new HashSet<>();
-        secondParameters.add(new Parameter(fooDescriptor, "baz"));
+        final Map<String, Parameter> secondParameters = new HashMap<>();
+        secondParameters.put("foo", new Parameter(fooDescriptor, "baz"));
         context.setParameters(secondParameters);
 
         assertTrue(context.getParameter("abc").isPresent());
         assertTrue(context.getParameter("xyz").isPresent());
 
-        secondParameters.add(new Parameter(abcParam.getDescriptor(), null));
-        secondParameters.add(new Parameter(xyzParam.getDescriptor(), null));
+        secondParameters.put("abc", null);
+        secondParameters.put("xyz", null);
 
         context.setParameters(secondParameters);
 
@@ -86,8 +85,8 @@ public class TestStandardParameterContext {
 
         assertEquals(Collections.singletonMap(fooDescriptor, fooParam), context.getParameters());
 
-        final Set<Parameter> thirdParameters = new HashSet<>();
-        thirdParameters.add(new Parameter(fooDescriptor, "other"));
+        final Map<String, Parameter> thirdParameters = new HashMap<>();
+        thirdParameters.put("foo", new Parameter(fooDescriptor, "other"));
         context.setParameters(thirdParameters);
 
         assertEquals("other", context.getParameter("foo").get().getValue());
@@ -100,8 +99,8 @@ public class TestStandardParameterContext {
 
         final ParameterDescriptor abcDescriptor = new ParameterDescriptor.Builder().name("abc").description("abc").build();
 
-        final Set<Parameter> parameters = new HashSet<>();
-        parameters.add(new Parameter(abcDescriptor, "123"));
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameters.put("abc", new Parameter(abcDescriptor, "123"));
 
         context.setParameters(parameters);
 
@@ -112,7 +111,7 @@ public class TestStandardParameterContext {
 
         ParameterDescriptor updatedDescriptor = new ParameterDescriptor.Builder().name("abc").description("Updated").build();
         final Parameter newDescriptionParam = new Parameter(updatedDescriptor, "321");
-        context.setParameters(Collections.singleton(newDescriptionParam));
+        context.setParameters(Collections.singletonMap("abc", newDescriptionParam));
 
         abcParam = context.getParameter("abc").get();
         assertEquals(abcDescriptor, abcParam.getDescriptor());
@@ -121,12 +120,12 @@ public class TestStandardParameterContext {
 
         updatedDescriptor = new ParameterDescriptor.Builder().name("abc").description("Updated Again").build();
         final Parameter paramWithoutValue = new Parameter(updatedDescriptor, null);
-        context.setParameters(Collections.singleton(paramWithoutValue));
+        context.setParameters(Collections.singletonMap("abc", paramWithoutValue));
 
         abcParam = context.getParameter("abc").get();
         assertEquals(abcDescriptor, abcParam.getDescriptor());
         assertEquals("Updated Again", abcParam.getDescriptor().getDescription());
-        assertEquals("321", abcParam.getValue());
+        assertNull(abcParam.getValue());
     }
 
     @Test
@@ -139,17 +138,17 @@ public class TestStandardParameterContext {
         final ParameterDescriptor xyzDescriptor = new ParameterDescriptor.Builder().name("xyz").build();
         final ParameterDescriptor fooDescriptor = new ParameterDescriptor.Builder().name("foo").description("bar").sensitive(true).build();
 
-        final Set<Parameter> parameters = new HashSet<>();
-        parameters.add(new Parameter(abcDescriptor, "123"));
-        parameters.add(new Parameter(xyzDescriptor, "242526"));
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameters.put("abc", new Parameter(abcDescriptor, "123"));
+        parameters.put("xyz", new Parameter(xyzDescriptor, "242526"));
 
         context.setParameters(parameters);
 
         final ParameterDescriptor sensitiveXyzDescriptor = new ParameterDescriptor.Builder().name("xyz").sensitive(true).build();
 
-        final Set<Parameter> updatedParameters = new HashSet<>();
-        updatedParameters.add(new Parameter(fooDescriptor, "baz"));
-        updatedParameters.add(new Parameter(sensitiveXyzDescriptor, "242526"));
+        final Map<String, Parameter> updatedParameters = new HashMap<>();
+        updatedParameters.put("foo", new Parameter(fooDescriptor, "baz"));
+        updatedParameters.put("xyz", new Parameter(sensitiveXyzDescriptor, "242526"));
 
         try {
             context.setParameters(updatedParameters);
@@ -159,7 +158,7 @@ public class TestStandardParameterContext {
 
         final ParameterDescriptor insensitiveAbcDescriptor = new ParameterDescriptor.Builder().name("abc").sensitive(false).build();
         updatedParameters.clear();
-        updatedParameters.add(new Parameter(insensitiveAbcDescriptor, "123"));
+        updatedParameters.put("abc", new Parameter(insensitiveAbcDescriptor, "123"));
 
         try {
             context.setParameters(updatedParameters);
@@ -179,13 +178,13 @@ public class TestStandardParameterContext {
 
         final ParameterDescriptor abcDescriptor = new ParameterDescriptor.Builder().name("abc").sensitive(true).build();
 
-        final Set<Parameter> parameters = new HashSet<>();
-        parameters.add(new Parameter(abcDescriptor, "123"));
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameters.put("abc", new Parameter(abcDescriptor, "123"));
 
         context.setParameters(parameters);
 
         parameters.clear();
-        parameters.add(new Parameter(abcDescriptor, "321"));
+        parameters.put("abc", new Parameter(abcDescriptor, "321"));
         context.setParameters(parameters);
 
         assertEquals("321", context.getParameter("abc").get().getValue());
@@ -194,7 +193,7 @@ public class TestStandardParameterContext {
         Mockito.when(procNode.isRunning()).thenReturn(true);
 
         parameters.clear();
-        parameters.add(new Parameter(abcDescriptor, "123"));
+        parameters.put("abc", new Parameter(abcDescriptor, "123"));
 
         try {
             context.setParameters(parameters);
@@ -202,10 +201,10 @@ public class TestStandardParameterContext {
         } catch (final IllegalStateException expected) {
         }
 
-        context.setParameters(Collections.emptySet());
+        context.setParameters(Collections.emptyMap());
 
         parameters.clear();
-        parameters.add(new Parameter(abcDescriptor, null));
+        parameters.put("abc", new Parameter(abcDescriptor, null));
         try {
             context.setParameters(parameters);
             Assert.fail("Was able to remove parameter while referencing processor was running");
@@ -224,15 +223,15 @@ public class TestStandardParameterContext {
         Mockito.when(serviceNode.getState()).thenReturn(ControllerServiceState.ENABLED);
 
         final ParameterDescriptor abcDescriptor = new ParameterDescriptor.Builder().name("abc").sensitive(true).build();
-        final Set<Parameter> parameters = new HashSet<>();
-        parameters.add(new Parameter(abcDescriptor, "123"));
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameters.put("abc", new Parameter(abcDescriptor, "123"));
 
         context.setParameters(parameters);
 
         referenceManager.addControllerServiceReference("abc", serviceNode);
 
         parameters.clear();
-        parameters.add(new Parameter(abcDescriptor, "321"));
+        parameters.put("abc", new Parameter(abcDescriptor, "321"));
 
         for (final ControllerServiceState state : EnumSet.of(ControllerServiceState.ENABLED, ControllerServiceState.ENABLING, ControllerServiceState.DISABLING)) {
             Mockito.when(serviceNode.getState()).thenReturn(state);
@@ -249,7 +248,7 @@ public class TestStandardParameterContext {
         parameters.clear();
         context.setParameters(parameters);
 
-        parameters.add(new Parameter(abcDescriptor, null));
+        parameters.put("abc", new Parameter(abcDescriptor, null));
         try {
             context.setParameters(parameters);
             Assert.fail("Was able to remove parameter being referenced by Controller Service that is DISABLING");
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/resources/int-tests/clustered-nifi.properties b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/resources/int-tests/clustered-nifi.properties
index 2516b42..809eaf3 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/resources/int-tests/clustered-nifi.properties
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/resources/int-tests/clustered-nifi.properties
@@ -218,7 +218,7 @@ nifi.cluster.flow.election.max.candidates=
 
 # cluster load balancing properties #
 nifi.cluster.load.balance.host=
-nifi.cluster.load.balance.port=6342
+nifi.cluster.load.balance.port=0
 nifi.cluster.load.balance.connections.per.node=4
 nifi.cluster.load.balance.max.thread.count=8
 nifi.cluster.load.balance.comms.timeout=30 sec
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java
index 34670e0..7594fa8 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java
@@ -1046,10 +1046,11 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
         final Set<ProcessGroup> boundProcessGroups = parameterContext.getParameterReferenceManager().getProcessGroupsBound(parameterContext);
 
         final ParameterContext updatedParameterContext = new StandardParameterContext(parameterContext.getIdentifier(), parameterContext.getName(), ParameterReferenceManager.EMPTY, null);
-        final Set<Parameter> parameters = parameterContextDto.getParameters().stream()
+        final Map<String, Parameter> parameters = new HashMap<>();
+        parameterContextDto.getParameters().stream()
             .map(ParameterEntity::getParameter)
             .map(this::createParameter)
-            .collect(Collectors.toSet());
+            .forEach(param -> parameters.put(param.getDescriptor().getName(), param));
         updatedParameterContext.setParameters(parameters);
 
         final List<ComponentValidationResultEntity> validationResults = new ArrayList<>();
@@ -1089,6 +1090,10 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
     }
 
     private Parameter createParameter(final ParameterDTO dto) {
+        if (dto.getDescription() == null && dto.getSensitive() == null && dto.getValue() == null) {
+            return null; // null description, sensitivity flag, and value indicates a deletion, which we want to represent as a null Parameter.
+        }
+
         final ParameterDescriptor descriptor = new ParameterDescriptor.Builder()
             .name(dto.getName())
             .description(dto.getDescription())
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/dao/impl/StandardParameterContextDAO.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/dao/impl/StandardParameterContextDAO.java
index aaab830..4b02b9d 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/dao/impl/StandardParameterContextDAO.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/dao/impl/StandardParameterContextDAO.java
@@ -37,10 +37,10 @@ import org.apache.nifi.web.api.entity.ParameterEntity;
 import org.apache.nifi.web.dao.ParameterContextDAO;
 
 import java.util.Collections;
+import java.util.HashMap;
 import java.util.List;
 import java.util.Map;
 import java.util.Set;
-import java.util.stream.Collectors;
 
 public class StandardParameterContextDAO implements ParameterContextDAO {
     private FlowManager flowManager;
@@ -57,7 +57,7 @@ public class StandardParameterContextDAO implements ParameterContextDAO {
 
     @Override
     public ParameterContext createParameterContext(final ParameterContextDTO parameterContextDto) {
-        final Set<Parameter> parameters = getParameters(parameterContextDto);
+        final Map<String, Parameter> parameters = getParameters(parameterContextDto);
         final ParameterContext parameterContext = flowManager.createParameterContext(parameterContextDto.getId(), parameterContextDto.getName(), parameters);
         if (parameterContextDto.getDescription() != null) {
             parameterContext.setDescription(parameterContextDto.getDescription());
@@ -65,16 +65,30 @@ public class StandardParameterContextDAO implements ParameterContextDAO {
         return parameterContext;
     }
 
-    private Set<Parameter> getParameters(final ParameterContextDTO parameterContextDto) {
-        final Set<ParameterEntity> parameterDtos = parameterContextDto.getParameters();
-        if (parameterDtos == null) {
-            return Collections.emptySet();
+    private Map<String, Parameter> getParameters(final ParameterContextDTO parameterContextDto) {
+        final Set<ParameterEntity> parameterEntities = parameterContextDto.getParameters();
+        if (parameterEntities == null) {
+            return Collections.emptyMap();
         }
 
-        return parameterContextDto.getParameters().stream()
-            .map(ParameterEntity::getParameter)
-            .map(this::createParameter)
-            .collect(Collectors.toSet());
+        final Map<String, Parameter> parameterMap = new HashMap<>();
+        for (final ParameterEntity parameterEntity : parameterEntities) {
+            final ParameterDTO parameterDto = parameterEntity.getParameter();
+
+            if (parameterDto.getName() == null) {
+                throw new IllegalArgumentException("Cannot specify a Parameter without a name");
+            }
+
+            final boolean deletion = parameterDto.getDescription() == null && parameterDto.getSensitive() == null && parameterDto.getValue() == null;
+            if (deletion) {
+                parameterMap.put(parameterDto.getName(), null);
+            } else {
+                final Parameter parameter = createParameter(parameterDto);
+                parameterMap.put(parameterDto.getName(), parameter);
+            }
+        }
+
+        return parameterMap;
     }
 
     private Parameter createParameter(final ParameterDTO dto) {
@@ -118,7 +132,7 @@ public class StandardParameterContextDAO implements ParameterContextDAO {
         }
 
         if (parameterContextDto.getParameters() != null) {
-            final Set<Parameter> parameters = getParameters(parameterContextDto);
+            final Map<String, Parameter> parameters = getParameters(parameterContextDto);
             context.setParameters(parameters);
         }
 
diff --git a/nifi-nar-bundles/nifi-scripting-bundle/nifi-scripting-processors/src/main/java/org/apache/nifi/script/impl/ValidationContextAdapter.java b/nifi-nar-bundles/nifi-scripting-bundle/nifi-scripting-processors/src/main/java/org/apache/nifi/script/impl/ValidationContextAdapter.java
index b983c16..eee7e37 100644
--- a/nifi-nar-bundles/nifi-scripting-bundle/nifi-scripting-processors/src/main/java/org/apache/nifi/script/impl/ValidationContextAdapter.java
+++ b/nifi-nar-bundles/nifi-scripting-bundle/nifi-scripting-processors/src/main/java/org/apache/nifi/script/impl/ValidationContextAdapter.java
@@ -101,6 +101,11 @@ public abstract class ValidationContextAdapter implements ValidationContext {
     }
 
     @Override
+    public boolean isParameterSet(final String parameterName) {
+        return innerValidationContext.isParameterSet(parameterName);
+    }
+
+    @Override
     public Collection<String> getReferencedParameters(final String propertyName) {
         return innerValidationContext.getReferencedParameters(propertyName);
     }
diff --git a/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessParameterContext.java b/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessParameterContext.java
index e48a965..2a9ad4f 100644
--- a/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessParameterContext.java
+++ b/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessParameterContext.java
@@ -61,13 +61,13 @@ public class StatelessParameterContext implements ParameterContext {
     }
 
     @Override
-    public void setParameters(final Set<Parameter> updatedParameters) {
-        throw new UnsupportedOperationException();
+    public void setParameters(final Map<String, Parameter> updatedParameters) {
+        throw new UnsupportedOperationException(); // This parameter context does not support updating - all parameters are provided in the constructor.
     }
 
     @Override
-    public void verifyCanSetParameters(final Set<Parameter> parameters) {
-        throw new UnsupportedOperationException();
+    public void verifyCanSetParameters(final Map<String, Parameter> parameters) {
+        throw new UnsupportedOperationException(); // This parameter context does not support updating - all parameters are provided in the constructor.
     }
 
     @Override
diff --git a/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessValidationContext.java b/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessValidationContext.java
index 81f1eb0..9ed8434 100644
--- a/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessValidationContext.java
+++ b/nifi-stateless/nifi-stateless-core/src/main/java/org/apache/nifi/stateless/core/StatelessValidationContext.java
@@ -26,6 +26,7 @@ import org.apache.nifi.controller.ControllerService;
 import org.apache.nifi.controller.ControllerServiceLookup;
 import org.apache.nifi.controller.PropertyConfiguration;
 import org.apache.nifi.expression.ExpressionLanguageCompiler;
+import org.apache.nifi.parameter.Parameter;
 import org.apache.nifi.parameter.ParameterContext;
 import org.apache.nifi.parameter.ParameterReference;
 import org.apache.nifi.registry.VariableRegistry;
@@ -36,6 +37,7 @@ import java.util.HashMap;
 import java.util.LinkedHashMap;
 import java.util.List;
 import java.util.Map;
+import java.util.Optional;
 import java.util.stream.Collectors;
 
 public class StatelessValidationContext implements ValidationContext {
@@ -147,6 +149,22 @@ public class StatelessValidationContext implements ValidationContext {
     }
 
     @Override
+    public boolean isParameterSet(final String parameterName) {
+        if (parameterContext == null) {
+            return false;
+        }
+
+        final Optional<Parameter> parameterOption = parameterContext.getParameter(parameterName);
+        if (!parameterOption.isPresent()) {
+            return false;
+        }
+
+        final String value = parameterOption.get().getValue();
+        return value != null;
+
+    }
+
+    @Override
     public ControllerServiceLookup getControllerServiceLookup() {
         return this.lookup;
     }


[nifi] 02/02: NIFI-6382: On flow import and version change, create parameter context where necessary and add parameters where necessary

Posted by bb...@apache.org.
This is an automated email from the ASF dual-hosted git repository.

bbende pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/nifi.git

commit bd8342c5e06e71846d1e5976bfe9de45690b974d
Author: Mark Payne <ma...@hotmail.com>
AuthorDate: Thu Aug 8 15:54:41 2019 -0400

    NIFI-6382: On flow import and version change, create parameter context where necessary and add parameters where necessary
    
    NIFI-6382: Fixed bugs reported in code review: sensitive properties referencing parameters should be exported to registry; when importing a versioned flow, any new parameters should be added to the parameter context
    
    NIFI-6382: Updated logic for handling parameter references with sensitive properties when integrating with nifi registry
    
    NIFI-6382: If user does not have permissions to READ a parameter context with a given name when importing a flow or changing flow version, ensure that NiFi properly generates an AccessDenied messages instead of throwing NullPointerException
    
    This closes #3642.
    
    Signed-off-by: Bryan Bende <bb...@apache.org>
---
 .../nifi/controller/AbstractComponentNode.java     |   2 +
 .../serialization/StandardFlowSerializer.java      |   3 +-
 .../apache/nifi/groups/StandardProcessGroup.java   | 181 ++++++++---
 .../nifi/groups/StandardVersionedFlowStatus.java   |   4 +-
 .../nifi/registry/flow/RestBasedFlowRegistry.java  |   2 +
 .../flow/mapping/NiFiRegistryFlowMapper.java       |  19 +-
 .../nifi/integration/FrameworkIntegrationTest.java |  11 +-
 .../integration/MockSingleFlowRegistryClient.java  | 205 ++++++++++++
 .../processors/UsernamePasswordProcessor.java      |  62 ++++
 .../nifi/integration/versioned/ImportFlowIT.java   | 343 ++++++++++++++++++++-
 .../authorization/AuthorizeParameterReference.java |  50 +++
 .../org/apache/nifi/web/NiFiServiceFacade.java     |  16 +-
 .../apache/nifi/web/StandardNiFiServiceFacade.java |  28 +-
 .../apache/nifi/web/api/ProcessGroupResource.java  |   9 +-
 .../org/apache/nifi/web/api/VersionsResource.java  |  17 +-
 .../org/apache/nifi/web/api/dto/DtoFactory.java    |   4 +-
 16 files changed, 901 insertions(+), 55 deletions(-)

diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java
index 48e359f..ea29d9e 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core-api/src/main/java/org/apache/nifi/controller/AbstractComponentNode.java
@@ -658,6 +658,8 @@ public abstract class AbstractComponentNode implements ComponentNode {
                         .valid(false)
                         .explanation("Property references Parameter '" + paramName + "' but the currently selected Parameter Context does not have a Parameter with that name")
                         .build());
+
+                    continue;
                 }
 
                 if (!validationContext.isParameterSet(paramName)) {
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/serialization/StandardFlowSerializer.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/serialization/StandardFlowSerializer.java
index a5c31b7..adedf3c 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/serialization/StandardFlowSerializer.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/serialization/StandardFlowSerializer.java
@@ -180,7 +180,8 @@ public class StandardFlowSerializer implements FlowSerializer<Document> {
         addStringElement(parameterElement, "sensitive", String.valueOf(descriptor.isSensitive()));
 
         if (descriptor.isSensitive()) {
-            addStringElement(parameterElement, "value", ENC_PREFIX + encryptor.encrypt(parameter.getValue()) + ENC_SUFFIX);
+            final String parameterValue = parameter.getValue();
+            addStringElement(parameterElement, "value", parameterValue == null ? null : ENC_PREFIX + encryptor.encrypt(parameterValue) + ENC_SUFFIX);
         } else {
             addStringElement(parameterElement, "value", parameter.getValue());
         }
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardProcessGroup.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardProcessGroup.java
index 526e172..52c5cbe 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardProcessGroup.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardProcessGroup.java
@@ -72,6 +72,7 @@ import org.apache.nifi.logging.LogRepositoryFactory;
 import org.apache.nifi.nar.NarCloseable;
 import org.apache.nifi.parameter.Parameter;
 import org.apache.nifi.parameter.ParameterContext;
+import org.apache.nifi.parameter.ParameterDescriptor;
 import org.apache.nifi.parameter.ParameterReference;
 import org.apache.nifi.processor.Relationship;
 import org.apache.nifi.processor.StandardProcessContext;
@@ -96,6 +97,8 @@ import org.apache.nifi.registry.flow.VersionedFlowState;
 import org.apache.nifi.registry.flow.VersionedFlowStatus;
 import org.apache.nifi.registry.flow.VersionedFunnel;
 import org.apache.nifi.registry.flow.VersionedLabel;
+import org.apache.nifi.registry.flow.VersionedParameter;
+import org.apache.nifi.registry.flow.VersionedParameterContext;
 import org.apache.nifi.registry.flow.VersionedPort;
 import org.apache.nifi.registry.flow.VersionedProcessGroup;
 import org.apache.nifi.registry.flow.VersionedProcessor;
@@ -3547,7 +3550,8 @@ public final class StandardProcessGroup implements ProcessGroup {
 
             final StandardVersionControlInformation originalVci = this.versionControlInfo.get();
             try {
-                updateProcessGroup(this, proposedSnapshot.getFlowContents(), componentIdSeed, updatedVersionedComponentIds, false, updateSettings, updateDescendantVersionedFlows, knownVariables);
+                updateProcessGroup(this, proposedSnapshot.getFlowContents(), componentIdSeed, updatedVersionedComponentIds, false, updateSettings, updateDescendantVersionedFlows, knownVariables,
+                    proposedSnapshot.getParameterContexts());
             } catch (final Throwable t) {
                 // The proposed snapshot may not have any Versioned Flow Coordinates. As a result, the call to #updateProcessGroup may
                 // set this PG's Version Control Info to null. During the normal flow of control,
@@ -3627,7 +3631,7 @@ public final class StandardProcessGroup implements ProcessGroup {
 
     private void updateProcessGroup(final ProcessGroup group, final VersionedProcessGroup proposed, final String componentIdSeed,
                                     final Set<String> updatedVersionedComponentIds, final boolean updatePosition, final boolean updateName, final boolean updateDescendantVersionedGroups,
-                                    final Set<String> variablesToSkip) throws ProcessorInstantiationException {
+                                    final Set<String> variablesToSkip, final Map<String, VersionedParameterContext> versionedParameterContexts) throws ProcessorInstantiationException {
 
         // During the flow update, we will use temporary names for process group ports. This is because port names must be
         // unique within a process group, but during an update we might temporarily be in a state where two ports have the same name.
@@ -3647,24 +3651,8 @@ public final class StandardProcessGroup implements ProcessGroup {
             group.setPosition(new Position(proposed.getPosition().getX(), proposed.getPosition().getY()));
         }
 
-        // Determine which variables have been added/removed and add/remove them from this group's variable registry.
-        // We don't worry about if a variable value has changed, because variables are designed to be 'environment specific.'
-        // As a result, once imported, we won't update variables to match the remote flow, but we will add any missing variables
-        // and remove any variables that are no longer part of the remote flow.
-        final Set<String> existingVariableNames = group.getVariableRegistry().getVariableMap().keySet().stream()
-                .map(VariableDescriptor::getName)
-                .collect(Collectors.toSet());
-
-        final Map<String, String> updatedVariableMap = new HashMap<>();
-
-        // If any new variables exist in the proposed flow, add those to the variable registry.
-        for (final Map.Entry<String, String> entry : proposed.getVariables().entrySet()) {
-            if (!existingVariableNames.contains(entry.getKey()) && !variablesToSkip.contains(entry.getKey())) {
-                updatedVariableMap.put(entry.getKey(), entry.getValue());
-            }
-        }
-
-        group.setVariables(updatedVariableMap);
+        updateParameterContext(group, proposed, versionedParameterContexts, componentIdSeed);
+        updateVariableRegistry(group, proposed, variablesToSkip);
 
         final VersionedFlowCoordinates remoteCoordinates = proposed.getVersionedFlowCoordinates();
         if (remoteCoordinates == null) {
@@ -3743,12 +3731,13 @@ public final class StandardProcessGroup implements ProcessGroup {
             final VersionedFlowCoordinates childCoordinates = proposedChildGroup.getVersionedFlowCoordinates();
 
             if (childGroup == null) {
-                final ProcessGroup added = addProcessGroup(group, proposedChildGroup, componentIdSeed, variablesToSkip);
+                final ProcessGroup added = addProcessGroup(group, proposedChildGroup, componentIdSeed, variablesToSkip, versionedParameterContexts);
                 flowManager.onProcessGroupAdded(added);
                 added.findAllRemoteProcessGroups().forEach(RemoteProcessGroup::initialize);
                 LOG.info("Added {} to {}", added, this);
             } else if (childCoordinates == null || updateDescendantVersionedGroups) {
-                updateProcessGroup(childGroup, proposedChildGroup, componentIdSeed, updatedVersionedComponentIds, true, true, updateDescendantVersionedGroups, variablesToSkip);
+                updateProcessGroup(childGroup, proposedChildGroup, componentIdSeed, updatedVersionedComponentIds, true, true, updateDescendantVersionedGroups,
+                    variablesToSkip, versionedParameterContexts);
                 LOG.info("Updated {}", childGroup);
             }
 
@@ -4017,6 +4006,102 @@ public final class StandardProcessGroup implements ProcessGroup {
         }
     }
 
+    private ParameterContext createParameterContext(final VersionedParameterContext versionedParameterContext, final String parameterContextId) {
+        final Map<String, Parameter> parameters = new HashMap<>();
+        for (final VersionedParameter versionedParameter : versionedParameterContext.getParameters()) {
+            final ParameterDescriptor descriptor = new ParameterDescriptor.Builder()
+                .name(versionedParameter.getName())
+                .description(versionedParameter.getDescription())
+                .sensitive(versionedParameter.isSensitive())
+                .build();
+
+            final Parameter parameter = new Parameter(descriptor, versionedParameter.getValue());
+            parameters.put(versionedParameter.getName(), parameter);
+        }
+
+        return flowController.getFlowManager().createParameterContext(parameterContextId, versionedParameterContext.getName(), parameters);
+    }
+
+    private void addMissingParameters(final VersionedParameterContext versionedParameterContext, final ParameterContext currentParameterContext) {
+        final Map<String, Parameter> parameters = new HashMap<>();
+        for (final VersionedParameter versionedParameter : versionedParameterContext.getParameters()) {
+            final Optional<Parameter> parameterOption = currentParameterContext.getParameter(versionedParameter.getName());
+            if (parameterOption.isPresent()) {
+                // Skip this parameter, since it is already defined. We only want to add missing parameters
+                continue;
+            }
+
+            final ParameterDescriptor descriptor = new ParameterDescriptor.Builder()
+                .name(versionedParameter.getName())
+                .description(versionedParameter.getDescription())
+                .sensitive(versionedParameter.isSensitive())
+                .build();
+
+            final Parameter parameter = new Parameter(descriptor, versionedParameter.getValue());
+            parameters.put(versionedParameter.getName(), parameter);
+        }
+
+        currentParameterContext.setParameters(parameters);
+    }
+
+    private ParameterContext getParameterContextByName(final String contextName) {
+        return flowController.getFlowManager().getParameterContextManager().getParameterContexts().stream()
+            .filter(context -> context.getName().equals(contextName))
+            .findAny()
+            .orElse(null);
+    }
+
+    private void updateParameterContext(final ProcessGroup group, final VersionedProcessGroup proposed, final Map<String, VersionedParameterContext> versionedParameterContexts,
+                                        final String componentIdSeed) {
+        // Update the Parameter Context
+        final ParameterContext currentParamContext = group.getParameterContext();
+        final String proposedParameterContextName = proposed.getParameterContextName();
+        if (proposedParameterContextName != null) {
+            if (currentParamContext == null) {
+                // Create a new Parameter Context based on the parameters provided
+                final VersionedParameterContext versionedParameterContext = versionedParameterContexts.get(proposedParameterContextName);
+
+                final ParameterContext contextByName = getParameterContextByName(versionedParameterContext.getName());
+                final ParameterContext selectedParameterContext;
+                if (contextByName == null) {
+                    final String parameterContextId = generateUuid(versionedParameterContext.getName(), versionedParameterContext.getName(), componentIdSeed);
+                    selectedParameterContext = createParameterContext(versionedParameterContext, parameterContextId);
+                } else {
+                    selectedParameterContext = contextByName;
+                    addMissingParameters(versionedParameterContext, selectedParameterContext);
+                }
+
+                group.setParameterContext(selectedParameterContext);
+            } else {
+                // Update the current Parameter Context so that it has any Parameters included in the proposed context
+                final VersionedParameterContext versionedParameterContext = versionedParameterContexts.get(proposedParameterContextName);
+                addMissingParameters(versionedParameterContext, currentParamContext);
+            }
+        }
+    }
+
+    private void updateVariableRegistry(final ProcessGroup group, final VersionedProcessGroup proposed, final Set<String> variablesToSkip) {
+        // Determine which variables have been added/removed and add/remove them from this group's variable registry.
+        // We don't worry about if a variable value has changed, because variables are designed to be 'environment specific.'
+        // As a result, once imported, we won't update variables to match the remote flow, but we will add any missing variables
+        // and remove any variables that are no longer part of the remote flow.
+        final Set<String> existingVariableNames = group.getVariableRegistry().getVariableMap().keySet().stream()
+            .map(VariableDescriptor::getName)
+            .collect(Collectors.toSet());
+
+        final Map<String, String> updatedVariableMap = new HashMap<>();
+
+        // If any new variables exist in the proposed flow, add those to the variable registry.
+        for (final Map.Entry<String, String> entry : proposed.getVariables().entrySet()) {
+            if (!existingVariableNames.contains(entry.getKey()) && !variablesToSkip.contains(entry.getKey())) {
+                updatedVariableMap.put(entry.getKey(), entry.getValue());
+            }
+        }
+
+        group.setVariables(updatedVariableMap);
+    }
+
+
     private String getPublicPortFinalName(final PublicPort publicPort, final String proposedFinalName) {
         final Optional<Port> existingPublicPort;
         if (TransferDirection.RECEIVE == publicPort.getDirection()) {
@@ -4074,12 +4159,13 @@ public final class StandardProcessGroup implements ProcessGroup {
     }
 
 
-    private ProcessGroup addProcessGroup(final ProcessGroup destination, final VersionedProcessGroup proposed, final String componentIdSeed, final Set<String> variablesToSkip)
+    private ProcessGroup addProcessGroup(final ProcessGroup destination, final VersionedProcessGroup proposed, final String componentIdSeed, final Set<String> variablesToSkip,
+                                         final Map<String, VersionedParameterContext> versionedParameterContexts)
             throws ProcessorInstantiationException {
         final ProcessGroup group = flowManager.createProcessGroup(generateUuid(proposed.getIdentifier(), destination.getIdentifier(), componentIdSeed));
         group.setVersionedComponentId(proposed.getIdentifier());
         group.setParent(destination);
-        updateProcessGroup(group, proposed, componentIdSeed, Collections.emptySet(), true, true, true, variablesToSkip);
+        updateProcessGroup(group, proposed, componentIdSeed, Collections.emptySet(), true, true, true, variablesToSkip, versionedParameterContexts);
         destination.addProcessGroup(group);
         return group;
     }
@@ -4308,7 +4394,7 @@ public final class StandardProcessGroup implements ProcessGroup {
             service.setComments(proposed.getComments());
             service.setName(proposed.getName());
 
-            final Map<String, String> properties = populatePropertiesMap(service.getEffectivePropertyValues(), proposed.getProperties(), proposed.getPropertyDescriptors(), service.getProcessGroup());
+            final Map<String, String> properties = populatePropertiesMap(service, proposed.getProperties(), proposed.getPropertyDescriptors(), service.getProcessGroup());
             service.setProperties(properties, true);
 
             if (!isEqual(service.getBundleCoordinate(), proposed.getBundle())) {
@@ -4447,7 +4533,7 @@ public final class StandardProcessGroup implements ProcessGroup {
             processor.setName(proposed.getName());
             processor.setPenalizationPeriod(proposed.getPenaltyDuration());
 
-            final Map<String, String> properties = populatePropertiesMap(processor.getRawPropertyValues(), proposed.getProperties(), proposed.getPropertyDescriptors(), processor.getProcessGroup());
+            final Map<String, String> properties = populatePropertiesMap(processor, proposed.getProperties(), proposed.getPropertyDescriptors(), processor.getProcessGroup());
             processor.setProperties(properties, true);
             processor.setRunDuration(proposed.getRunDurationMillis(), TimeUnit.MILLISECONDS);
             processor.setSchedulingStrategy(SchedulingStrategy.valueOf(proposed.getSchedulingStrategy()));
@@ -4470,7 +4556,7 @@ public final class StandardProcessGroup implements ProcessGroup {
     }
 
 
-    private Map<String, String> populatePropertiesMap(final Map<PropertyDescriptor, String> currentProperties, final Map<String, String> proposedProperties,
+    private Map<String, String> populatePropertiesMap(final ComponentNode componentNode, final Map<String, String> proposedProperties,
                                                       final Map<String, VersionedPropertyDescriptor> proposedDescriptors, final ProcessGroup group) {
 
         // since VersionedPropertyDescriptor currently doesn't know if it is sensitive or not,
@@ -4478,7 +4564,7 @@ public final class StandardProcessGroup implements ProcessGroup {
         final Set<String> sensitiveProperties = new HashSet<>();
 
         final Map<String, String> fullPropertyMap = new HashMap<>();
-        for (final PropertyDescriptor property : currentProperties.keySet()) {
+        for (final PropertyDescriptor property : componentNode.getRawPropertyValues().keySet()) {
             if (property.isSensitive()) {
                 sensitiveProperties.add(property.getName());
             } else {
@@ -4487,25 +4573,46 @@ public final class StandardProcessGroup implements ProcessGroup {
         }
 
         if (proposedProperties != null) {
-            for (final Map.Entry<String, String> entry : proposedProperties.entrySet()) {
-                final String propertyName = entry.getKey();
+            // Build a Set of all properties that are included in either the currently configured property values or the proposed values.
+            final Set<String> updatedPropertyNames = new HashSet<>();
+            updatedPropertyNames.addAll(proposedProperties.keySet());
+            componentNode.getProperties().keySet().stream()
+                .map(PropertyDescriptor::getName)
+                .forEach(updatedPropertyNames::add);
+
+            for (final String propertyName : updatedPropertyNames) {
                 final VersionedPropertyDescriptor descriptor = proposedDescriptors.get(propertyName);
 
-                // skip any sensitive properties so we can retain whatever is currently set
-                if (sensitiveProperties.contains(propertyName)) {
-                    continue;
-                }
-
                 String value;
                 if (descriptor != null && descriptor.getIdentifiesControllerService()) {
                     // Property identifies a Controller Service. So the value that we want to assign is not the value given.
                     // The value given is instead the Versioned Component ID of the Controller Service. We want to resolve this
                     // to the instance ID of the Controller Service.
-                    final String serviceVersionedComponentId = entry.getValue();
+                    final String serviceVersionedComponentId = proposedProperties.get(propertyName);
                     String instanceId = getServiceInstanceId(serviceVersionedComponentId, group);
                     value = instanceId == null ? serviceVersionedComponentId : instanceId;
                 } else {
-                    value = entry.getValue();
+                    value = proposedProperties.get(propertyName);
+                }
+
+                // skip any sensitive properties that are not populated so we can retain whatever is currently set. We do this because sensitive properties are not stored in the registry
+                // unless the value is a reference to a Parameter. If the value in the registry is null, it indicates that the sensitive value was removed, so we want to keep the currently
+                // populated value. The exception to this rule is if the currently configured value is a Parameter Reference and the Versioned Flow is empty. In this case, it implies
+                // that the Versioned Flow has changed from a Parameter Reference to an explicit value. In this case, we do in fact want to change the value of the Sensitive Property from
+                // the current parameter reference to an unset value.
+                if (sensitiveProperties.contains(propertyName) && value == null) {
+                    final PropertyConfiguration propertyConfiguration = componentNode.getProperty(componentNode.getPropertyDescriptor(propertyName));
+                    if (propertyConfiguration == null) {
+                        continue;
+                    }
+
+                    // No parameter references. Property currently is set to an explicit value. We don't want to change it.
+                    if (propertyConfiguration.getParameterReferences().isEmpty()) {
+                        continue;
+                    }
+
+                    // Once we reach this point, the property is configured to reference a Parameter, and the value in the Versioned Flow is an explicit value,
+                    // so we want to continue on and update the value to null.
                 }
 
                 fullPropertyMap.put(propertyName, value);
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardVersionedFlowStatus.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardVersionedFlowStatus.java
index 4be9898..604ec42 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardVersionedFlowStatus.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/groups/StandardVersionedFlowStatus.java
@@ -20,11 +20,11 @@ package org.apache.nifi.groups;
 import org.apache.nifi.registry.flow.VersionedFlowState;
 import org.apache.nifi.registry.flow.VersionedFlowStatus;
 
-class StandardVersionedFlowStatus implements VersionedFlowStatus {
+public class StandardVersionedFlowStatus implements VersionedFlowStatus {
     private final VersionedFlowState state;
     private final String explanation;
 
-    StandardVersionedFlowStatus(final VersionedFlowState state, final String explanation) {
+    public StandardVersionedFlowStatus(final VersionedFlowState state, final String explanation) {
         this.state = state;
         this.explanation = explanation;
     }
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/RestBasedFlowRegistry.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/RestBasedFlowRegistry.java
index 08168b1..a7f87d2 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/RestBasedFlowRegistry.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/RestBasedFlowRegistry.java
@@ -36,6 +36,7 @@ import java.util.Set;
 import java.util.stream.Collectors;
 
 public class RestBasedFlowRegistry implements FlowRegistry {
+    private static final String FLOW_ENCODING_VERSION = "1.0";
 
     private final FlowRegistryClient flowRegistryClient;
     private final String identifier;
@@ -189,6 +190,7 @@ public class RestBasedFlowRegistry implements FlowRegistry {
         versionedFlowSnapshot.setFlowContents(snapshot);
         versionedFlowSnapshot.setExternalControllerServices(externalControllerServices);
         versionedFlowSnapshot.setParameterContexts(parameterContextMap);
+        versionedFlowSnapshot.setFlowEncodingVersion(FLOW_ENCODING_VERSION);
 
         final VersionedFlowSnapshotMetadata metadata = new VersionedFlowSnapshotMetadata();
         metadata.setBucketIdentifier(flow.getBucketIdentifier());
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/mapping/NiFiRegistryFlowMapper.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/mapping/NiFiRegistryFlowMapper.java
index 3c0e8f2..fdfc143 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/mapping/NiFiRegistryFlowMapper.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/registry/flow/mapping/NiFiRegistryFlowMapper.java
@@ -27,6 +27,7 @@ import org.apache.nifi.connectable.Port;
 import org.apache.nifi.controller.ComponentNode;
 import org.apache.nifi.controller.ControllerService;
 import org.apache.nifi.controller.ProcessorNode;
+import org.apache.nifi.controller.PropertyConfiguration;
 import org.apache.nifi.controller.label.Label;
 import org.apache.nifi.controller.queue.FlowFileQueue;
 import org.apache.nifi.controller.service.ControllerServiceNode;
@@ -362,7 +363,7 @@ public class NiFiRegistryFlowMapper {
         final Map<String, String> mapped = new HashMap<>();
 
         component.getProperties().keySet().stream()
-            .filter(property -> !property.isSensitive())
+            .filter(property -> isMappable(property, component.getProperty(property)))
             .forEach(property -> {
                 String value = component.getRawPropertyValue(property);
                 if (value == null) {
@@ -384,6 +385,21 @@ public class NiFiRegistryFlowMapper {
         return mapped;
     }
 
+    private boolean isMappable(final PropertyDescriptor propertyDescriptor, final PropertyConfiguration propertyConfiguration) {
+        if (!propertyDescriptor.isSensitive()) { // If the property is not sensitive, it can be mapped.
+            return true;
+        }
+
+        if (propertyConfiguration == null) {
+            return false;
+        }
+
+        // Sensitive properties can be mapped if and only if they reference a Parameter. If a sensitive property references a parameter, it cannot contain any other value around it.
+        // For example, for a non-sensitive property, a value of "hello#{param}123" is valid, but for a sensitive property, it is invalid. Only something like "hello123" or "#{param}" is valid.
+        // Thus, we will map sensitive properties only if they reference a parameter.
+        return !propertyConfiguration.getParameterReferences().isEmpty();
+    }
+
     private Map<String, VersionedPropertyDescriptor> mapPropertyDescriptors(final ComponentNode component, final ControllerServiceProvider serviceProvider, final Set<String> includedGroupIds,
                                                                             final Map<String, ExternalControllerServiceReference> externalControllerServiceReferences) {
         final Map<String, VersionedPropertyDescriptor> descriptors = new HashMap<>();
@@ -391,6 +407,7 @@ public class NiFiRegistryFlowMapper {
             final VersionedPropertyDescriptor versionedDescriptor = new VersionedPropertyDescriptor();
             versionedDescriptor.setName(descriptor.getName());
             versionedDescriptor.setDisplayName(descriptor.getDisplayName());
+            versionedDescriptor.setSensitive(descriptor.isSensitive());
 
             final Class<?> referencedServiceType = descriptor.getControllerServiceDefinition();
             versionedDescriptor.setIdentifiesControllerService(referencedServiceType != null);
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/FrameworkIntegrationTest.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/FrameworkIntegrationTest.java
index 6826b59..8a01779 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/FrameworkIntegrationTest.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/FrameworkIntegrationTest.java
@@ -76,6 +76,7 @@ import org.apache.nifi.integration.processors.GenerateProcessor;
 import org.apache.nifi.integration.processors.NopProcessor;
 import org.apache.nifi.integration.processors.TerminateAll;
 import org.apache.nifi.integration.processors.TerminateOnce;
+import org.apache.nifi.integration.processors.UsernamePasswordProcessor;
 import org.apache.nifi.logging.LogRepositoryFactory;
 import org.apache.nifi.nar.ExtensionManager;
 import org.apache.nifi.nar.SystemBundle;
@@ -143,7 +144,7 @@ public class FrameworkIntegrationTest {
 
     private FlowEngine flowEngine;
     private FlowController flowController;
-    private FlowRegistryClient flowRegistryClient = new StandardFlowRegistryClient();
+    private FlowRegistryClient flowRegistryClient = createFlowRegistryClient();
     private ProcessorNode nopProcessor;
     private ProcessorNode terminateProcessor;
     private ProcessorNode terminateAllProcessor;
@@ -193,6 +194,13 @@ public class FrameworkIntegrationTest {
         initialize(nifiProperties);
     }
 
+    /**
+     * This method exists for subclasses to override and return a different implementation.
+     */
+    protected FlowRegistryClient createFlowRegistryClient() {
+        return new StandardFlowRegistryClient();
+    }
+
     protected final void initialize(final NiFiProperties nifiProperties) throws IOException {
         this.nifiProperties = nifiProperties;
 
@@ -214,6 +222,7 @@ public class FrameworkIntegrationTest {
         extensionManager.injectExtensionType(Processor.class, TerminateOnce.class);
         extensionManager.injectExtensionType(Processor.class, TerminateAll.class);
         extensionManager.injectExtensionType(Processor.class, NopProcessor.class);
+        extensionManager.injectExtensionType(Processor.class, UsernamePasswordProcessor.class);
 
         injectExtensionTypes(extensionManager);
         systemBundle = SystemBundle.create(nifiProperties);
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/MockSingleFlowRegistryClient.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/MockSingleFlowRegistryClient.java
new file mode 100644
index 0000000..0a6c932
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/MockSingleFlowRegistryClient.java
@@ -0,0 +1,205 @@
+/*
+ * 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.integration;
+
+import org.apache.nifi.authorization.user.NiFiUser;
+import org.apache.nifi.registry.bucket.Bucket;
+import org.apache.nifi.registry.client.NiFiRegistryException;
+import org.apache.nifi.registry.flow.ExternalControllerServiceReference;
+import org.apache.nifi.registry.flow.FlowRegistry;
+import org.apache.nifi.registry.flow.FlowRegistryClient;
+import org.apache.nifi.registry.flow.VersionedFlow;
+import org.apache.nifi.registry.flow.VersionedFlowSnapshot;
+import org.apache.nifi.registry.flow.VersionedFlowSnapshotMetadata;
+import org.apache.nifi.registry.flow.VersionedParameterContext;
+import org.apache.nifi.registry.flow.VersionedProcessGroup;
+
+import java.io.IOException;
+import java.util.Collection;
+import java.util.Collections;
+import java.util.Map;
+import java.util.Set;
+import java.util.concurrent.ConcurrentHashMap;
+
+public class MockSingleFlowRegistryClient implements FlowRegistryClient {
+    private final MockFlowRegistry flowRegistry = new MockFlowRegistry();
+
+    @Override
+    public FlowRegistry getFlowRegistry(final String registryId) {
+        return flowRegistry;
+    }
+
+    @Override
+    public Set<String> getRegistryIdentifiers() {
+        return Collections.singleton("unit-test-registry-client-id");
+    }
+
+    @Override
+    public void addFlowRegistry(final FlowRegistry registry) {
+
+    }
+
+    @Override
+    public FlowRegistry addFlowRegistry(final String registryId, final String registryName, final String registryUrl, final String description) {
+        return null;
+    }
+
+    @Override
+    public FlowRegistry removeFlowRegistry(final String registryId) {
+        return null;
+    }
+
+    public void addFlow(final String bucketId, final String flowId, final int version, final VersionedFlowSnapshot snapshot) {
+        flowRegistry.addFlow(bucketId, flowId, version, snapshot);
+    }
+
+
+    private static class FlowCoordinates {
+        private final String bucketId;
+        private final String flowId;
+        private final int version;
+
+        public FlowCoordinates(final String bucketId, final String flowId, final int version) {
+            this.bucketId = bucketId;
+            this.flowId = flowId;
+            this.version = version;
+        }
+
+        public String getBucketId() {
+            return bucketId;
+        }
+
+        public String getFlowId() {
+            return flowId;
+        }
+
+        public int getVersion() {
+            return version;
+        }
+    }
+
+
+    public static class MockFlowRegistry implements FlowRegistry {
+        private final Map<FlowCoordinates, VersionedFlowSnapshot> snapshots = new ConcurrentHashMap<>();
+
+        public void addFlow(final String bucketId, final String flowId, final int version, final VersionedFlowSnapshot snapshot) {
+            final FlowCoordinates coordinates = new FlowCoordinates(bucketId, flowId, version);
+            snapshots.put(coordinates, snapshot);
+        }
+
+
+        @Override
+        public String getIdentifier() {
+            return "int-test-flow-registry";
+        }
+
+        @Override
+        public String getDescription() {
+            return null;
+        }
+
+        @Override
+        public void setDescription(final String description) {
+
+        }
+
+        @Override
+        public String getURL() {
+            return "http://localhost:18080/integration-test";
+        }
+
+        @Override
+        public void setURL(final String url) {
+
+        }
+
+        @Override
+        public String getName() {
+            return "Integration Test Registry";
+        }
+
+        @Override
+        public void setName(final String name) {
+
+        }
+
+        @Override
+        public Set<Bucket> getBuckets(final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public Bucket getBucket(final String bucketId, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public Set<VersionedFlow> getFlows(final String bucketId, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public Set<VersionedFlowSnapshotMetadata> getFlowVersions(final String bucketId, final String flowId, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public VersionedFlow registerVersionedFlow(final VersionedFlow flow, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public VersionedFlow deleteVersionedFlow(final String bucketId, final String flowId, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public VersionedFlowSnapshot registerVersionedFlowSnapshot(final VersionedFlow flow, final VersionedProcessGroup snapshot,
+                                                                   final Map<String, ExternalControllerServiceReference> externalControllerServices,
+                                                                   final Collection<VersionedParameterContext> parameterContexts, final String comments,
+                                                                   final int expectedVersion, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public int getLatestVersion(final String bucketId, final String flowId, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return 0;
+        }
+
+        @Override
+        public VersionedFlowSnapshot getFlowContents(final String bucketId, final String flowId, final int version, final boolean fetchRemoteFlows, final NiFiUser user)
+                        throws IOException, NiFiRegistryException {
+            return getFlowContents(bucketId, flowId, version, fetchRemoteFlows);
+        }
+
+        @Override
+        public VersionedFlowSnapshot getFlowContents(final String bucketId, final String flowId, final int version, final boolean fetchRemoteFlows) throws IOException, NiFiRegistryException {
+            final FlowCoordinates coordinates = new FlowCoordinates(bucketId, flowId, version);
+            return snapshots.get(coordinates);
+        }
+
+        @Override
+        public VersionedFlow getVersionedFlow(final String bucketId, final String flowId, final NiFiUser user) throws IOException, NiFiRegistryException {
+            return null;
+        }
+
+        @Override
+        public VersionedFlow getVersionedFlow(final String bucketId, final String flowId) throws IOException, NiFiRegistryException {
+            return null;
+        }
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/processors/UsernamePasswordProcessor.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/processors/UsernamePasswordProcessor.java
new file mode 100644
index 0000000..09378e0
--- /dev/null
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/processors/UsernamePasswordProcessor.java
@@ -0,0 +1,62 @@
+/*
+ * 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.integration.processors;
+
+import org.apache.nifi.components.PropertyDescriptor;
+import org.apache.nifi.components.PropertyDescriptor.Builder;
+import org.apache.nifi.expression.ExpressionLanguageScope;
+import org.apache.nifi.processor.AbstractProcessor;
+import org.apache.nifi.processor.ProcessContext;
+import org.apache.nifi.processor.ProcessSession;
+import org.apache.nifi.processor.exception.ProcessException;
+import org.apache.nifi.processor.util.StandardValidators;
+
+import java.util.Arrays;
+import java.util.List;
+
+public class UsernamePasswordProcessor extends AbstractProcessor {
+    public static final PropertyDescriptor USERNAME = new Builder()
+        .name("username")
+        .displayName("username")
+        .description("username")
+        .required(false)
+        .addValidator(StandardValidators.NON_EMPTY_VALIDATOR)
+        .expressionLanguageSupported(ExpressionLanguageScope.FLOWFILE_ATTRIBUTES)
+        .defaultValue("")
+        .build();
+
+    public static final PropertyDescriptor PASSWORD = new Builder()
+        .name("password")
+        .displayName("password")
+        .description("password")
+        .required(false)
+        .sensitive(true)
+        .addValidator(StandardValidators.NON_EMPTY_VALIDATOR)
+        .expressionLanguageSupported(ExpressionLanguageScope.FLOWFILE_ATTRIBUTES)
+        .defaultValue("")
+        .build();
+
+    @Override
+    protected List<PropertyDescriptor> getSupportedPropertyDescriptors() {
+        return Arrays.asList(USERNAME, PASSWORD);
+    }
+
+    @Override
+    public void onTrigger(final ProcessContext context, final ProcessSession session) throws ProcessException {
+
+    }
+}
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java
index 99f91b8..996d363 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/test/java/org/apache/nifi/integration/versioned/ImportFlowIT.java
@@ -27,6 +27,13 @@ import org.apache.nifi.integration.DirectInjectionExtensionManager;
 import org.apache.nifi.integration.FrameworkIntegrationTest;
 import org.apache.nifi.integration.cs.LongValidatingControllerService;
 import org.apache.nifi.integration.cs.NopServiceReferencingProcessor;
+import org.apache.nifi.integration.processors.UsernamePasswordProcessor;
+import org.apache.nifi.parameter.Parameter;
+import org.apache.nifi.parameter.ParameterContext;
+import org.apache.nifi.parameter.ParameterDescriptor;
+import org.apache.nifi.parameter.ParameterReferenceManager;
+import org.apache.nifi.parameter.StandardParameterContext;
+import org.apache.nifi.parameter.StandardParameterReferenceManager;
 import org.apache.nifi.processor.Processor;
 import org.apache.nifi.registry.bucket.Bucket;
 import org.apache.nifi.registry.flow.Bundle;
@@ -38,21 +45,36 @@ import org.apache.nifi.registry.flow.VersionedParameter;
 import org.apache.nifi.registry.flow.VersionedParameterContext;
 import org.apache.nifi.registry.flow.VersionedProcessGroup;
 import org.apache.nifi.registry.flow.VersionedProcessor;
+import org.apache.nifi.registry.flow.diff.ComparableDataFlow;
+import org.apache.nifi.registry.flow.diff.ConciseEvolvingDifferenceDescriptor;
+import org.apache.nifi.registry.flow.diff.DifferenceType;
+import org.apache.nifi.registry.flow.diff.FlowComparator;
+import org.apache.nifi.registry.flow.diff.FlowComparison;
+import org.apache.nifi.registry.flow.diff.FlowDifference;
+import org.apache.nifi.registry.flow.diff.StandardComparableDataFlow;
+import org.apache.nifi.registry.flow.diff.StandardFlowComparator;
 import org.apache.nifi.registry.flow.mapping.NiFiRegistryFlowMapper;
+import org.apache.nifi.util.FlowDifferenceFilters;
 import org.junit.Test;
 
+import java.nio.charset.StandardCharsets;
 import java.util.Collection;
 import java.util.Collections;
 import java.util.HashMap;
 import java.util.HashSet;
 import java.util.List;
 import java.util.Map;
+import java.util.Optional;
 import java.util.Set;
+import java.util.UUID;
 import java.util.concurrent.ExecutionException;
+import java.util.stream.Collectors;
 
 import static junit.framework.TestCase.assertTrue;
 import static org.junit.Assert.assertEquals;
 import static org.junit.Assert.assertNotEquals;
+import static org.testng.Assert.assertNotNull;
+import static org.testng.AssertJUnit.assertNull;
 
 public class ImportFlowIT extends FrameworkIntegrationTest {
 
@@ -110,7 +132,312 @@ public class ImportFlowIT extends FrameworkIntegrationTest {
     }
 
 
-    private VersionedFlowSnapshot createFlowSnapshot(final List<ControllerServiceNode> controllerServices, final List<ProcessorNode> processors, final Map<String, String> parameters) {
+    @Test
+    public void testLocalModificationWhenSensitivePropReferencesParameter() {
+        // Create a processor with a sensitive property
+        final ProcessorNode processor = createProcessorNode(UsernamePasswordProcessor.class);
+        processor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "password"));
+
+        // Create a VersionedFlowSnapshot that contains the processor
+        final VersionedFlowSnapshot versionedFlowWithExplicitValue = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(processor), null);
+
+        // Create child group
+        final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
+        innerGroup.setName("Inner Group");
+        getRootGroup().addProcessGroup(innerGroup);
+
+        // Move processor into the child group
+        moveProcessor(processor, innerGroup);
+
+        // Verify that there are no differences between the versioned flow and the Process Group
+        Set<FlowDifference> differences = getLocalModifications(innerGroup, versionedFlowWithExplicitValue);
+        assertEquals(0, differences.size());
+
+        // Change the value of the sensitive property from one explicit value to another. Verify no local modifications.
+        processor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "secret"));
+        differences = getLocalModifications(innerGroup, versionedFlowWithExplicitValue);
+        assertEquals(0, differences.size());
+
+        // Change the value of the sensitive property to now reference a parameter. There should be one local modification.
+        processor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "#{secret-parameter}"));
+        differences = getLocalModifications(innerGroup, versionedFlowWithExplicitValue);
+        assertEquals(1, differences.size());
+        assertEquals(DifferenceType.PROPERTY_ADDED, differences.iterator().next().getDifferenceType());
+
+        // Create a Versioned Flow that contains the Parameter Reference.
+        final VersionedFlowSnapshot versionedFlowWithParameterReference = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(processor), null);
+
+        // Ensure no difference between the current configuration and the versioned flow
+        differences = getLocalModifications(innerGroup, versionedFlowWithParameterReference);
+        assertEquals(0, differences.size());
+
+        processor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "secret"));
+        differences = getLocalModifications(innerGroup, versionedFlowWithParameterReference);
+        assertEquals(1, differences.size());
+        assertEquals(DifferenceType.PROPERTY_REMOVED, differences.iterator().next().getDifferenceType());
+    }
+
+    @Test
+    public void testParameterCreatedWithNullValueOnImportWithSensitivePropertyReference() {
+        // Create a processor with a sensitive property
+        final ProcessorNode processor = createProcessorNode(UsernamePasswordProcessor.class);
+        processor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "#{secret-param}"));
+
+        // Create a VersionedFlowSnapshot that contains the processor
+        final Parameter parameter = new Parameter(new ParameterDescriptor.Builder().name("secret-param").sensitive(true).build(), null);
+        final VersionedFlowSnapshot versionedFlowWithParameterReference = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(processor), Collections.singleton(parameter));
+
+        // Create child group
+        final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
+        innerGroup.setName("Inner Group");
+        getRootGroup().addProcessGroup(innerGroup);
+
+        final ParameterReferenceManager parameterReferenceManager = new StandardParameterReferenceManager(getFlowController().getFlowManager());
+        final ParameterContext parameterContext = new StandardParameterContext("param-context-id", "parameter-context", parameterReferenceManager, null);
+        innerGroup.setParameterContext(parameterContext);
+
+        assertTrue(parameterContext.getParameters().isEmpty());
+
+        innerGroup.updateFlow(versionedFlowWithParameterReference, null, true, true, true);
+
+        final Collection<Parameter> parameters = parameterContext.getParameters().values();
+        assertEquals(1, parameters.size());
+
+        final Parameter firstParameter = parameters.iterator().next();
+        assertEquals("secret-param", firstParameter.getDescriptor().getName());
+        assertTrue(firstParameter.getDescriptor().isSensitive());
+        assertNull(firstParameter.getValue());
+    }
+
+    @Test
+    public void testParameterContextCreatedOnImportWithSensitivePropertyReference() {
+        // Create a processor with a sensitive property
+        final ProcessorNode processor = createProcessorNode(UsernamePasswordProcessor.class);
+        processor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "#{secret-param}"));
+
+        // Create a VersionedFlowSnapshot that contains the processor
+        final Parameter parameter = new Parameter(new ParameterDescriptor.Builder().name("secret-param").sensitive(true).build(), null);
+        final VersionedFlowSnapshot versionedFlowWithParameterReference = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(processor), Collections.singleton(parameter));
+
+        // Create child group
+        final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
+        innerGroup.setName("Inner Group");
+        getRootGroup().addProcessGroup(innerGroup);
+
+        innerGroup.updateFlow(versionedFlowWithParameterReference, null, true, true, true);
+
+        final ParameterContext parameterContext = innerGroup.getParameterContext();
+        assertNotNull(parameterContext);
+
+        final Collection<Parameter> parameters = parameterContext.getParameters().values();
+        assertEquals(1, parameters.size());
+
+        final Parameter firstParameter = parameters.iterator().next();
+        assertEquals("secret-param", firstParameter.getDescriptor().getName());
+        assertTrue(firstParameter.getDescriptor().isSensitive());
+        assertNull(firstParameter.getValue());
+    }
+
+
+    @Test
+    public void testChangeVersionFromParameterToExplicitValueSensitiveProperty() {
+        // Create a processor with a sensitive property
+        final ProcessorNode initialProcessor = createProcessorNode(UsernamePasswordProcessor.class);
+        initialProcessor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "#{secret-param}"));
+
+        // Create a VersionedFlowSnapshot that contains the processor
+        final Parameter parameter = new Parameter(new ParameterDescriptor.Builder().name("secret-param").sensitive(true).build(), null);
+        final VersionedFlowSnapshot versionedFlowWithParameterReference = createFlowSnapshot(Collections.emptyList(),
+            Collections.singletonList(initialProcessor), Collections.singleton(parameter));
+
+
+        // Update processor to have an explicit value for the second version of the flow.
+        initialProcessor.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "secret-value"));
+        final VersionedFlowSnapshot versionedFlowExplicitValue = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(initialProcessor), null);
+
+        // Create child group and update to the first version of the flow, with parameter ref
+        final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
+        innerGroup.setName("Inner Group");
+        getRootGroup().addProcessGroup(innerGroup);
+
+        innerGroup.updateFlow(versionedFlowWithParameterReference, null, true, true, true);
+
+        final ProcessorNode nodeInGroupWithRef = innerGroup.getProcessors().iterator().next();
+        assertNotNull(nodeInGroupWithRef.getProperty(UsernamePasswordProcessor.PASSWORD).getRawValue());
+
+        // Update the flow to new version that uses explicit value.
+        innerGroup.updateFlow(versionedFlowExplicitValue, null, true, true, true);
+
+        // Updated flow has sensitive property that no longer references parameter. Now is an explicit value, so it should be unset
+        final ProcessorNode nodeInGroupWithNoValue = innerGroup.getProcessors().iterator().next();
+        assertNull(nodeInGroupWithNoValue.getProperty(UsernamePasswordProcessor.PASSWORD).getRawValue());
+    }
+
+    @Test
+    public void testChangeVersionFromExplicitToExplicitValueDoesNotChangeSensitiveProperty() {
+        // Create a processor with a sensitive property and create a versioned flow for it.
+        final ProcessorNode initialProcessor = createProcessorNode(UsernamePasswordProcessor.class);
+        final Map<String, String> initialProperties = new HashMap<>();
+        initialProperties.put(UsernamePasswordProcessor.USERNAME.getName(), "user");
+        initialProperties.put(UsernamePasswordProcessor.PASSWORD.getName(), "pass");
+        initialProcessor.setProperties(initialProperties);
+
+        final VersionedFlowSnapshot initialVersionSnapshot = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(initialProcessor), null);
+
+        // Update processor to have a different explicit value for both sensitive and non-sensitive properties and create a versioned flow for it.
+        final Map<String, String> updatedProperties = new HashMap<>();
+        updatedProperties.put(UsernamePasswordProcessor.USERNAME.getName(), "other");
+        updatedProperties.put(UsernamePasswordProcessor.PASSWORD.getName(), "pass");
+        initialProcessor.setProperties(updatedProperties);
+
+        final VersionedFlowSnapshot updatedVersionSnapshot = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(initialProcessor), null);
+
+        // Create child group and update to the first version of the flow, with parameter ref
+        final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
+        innerGroup.setName("Inner Group");
+        getRootGroup().addProcessGroup(innerGroup);
+
+        // Import the flow into our newly created group
+        innerGroup.updateFlow(initialVersionSnapshot, null, true, true, true);
+
+        final ProcessorNode initialImportedProcessor = innerGroup.getProcessors().iterator().next();
+        assertEquals("user", initialImportedProcessor.getProperty(UsernamePasswordProcessor.USERNAME).getRawValue());
+        assertNull("pass", initialImportedProcessor.getProperty(UsernamePasswordProcessor.PASSWORD).getRawValue());
+
+        // Update the sensitive property to "pass"
+        initialImportedProcessor.setProperties(initialProperties);
+        assertEquals("pass", initialImportedProcessor.getProperty(UsernamePasswordProcessor.PASSWORD).getRawValue());
+
+        // Update the flow to new version
+        innerGroup.updateFlow(updatedVersionSnapshot, null, true, true, true);
+
+        // Updated flow has sensitive property that no longer references parameter. Now is an explicit value, so it should be unset
+        final ProcessorNode updatedImportedProcessor = innerGroup.getProcessors().iterator().next();
+        assertEquals("other", updatedImportedProcessor.getProperty(UsernamePasswordProcessor.USERNAME).getRawValue());
+        assertEquals("pass", updatedImportedProcessor.getProperty(UsernamePasswordProcessor.PASSWORD).getRawValue());
+    }
+
+
+    @Test
+    public void testChangeVersionFromParamReferenceToAnotherParamReferenceIsLocalModification() {
+        // Create a processor with a sensitive property and create a versioned flow for it.
+        final ProcessorNode initialProcessor = createProcessorNode(UsernamePasswordProcessor.class);
+        final Map<String, String> initialProperties = new HashMap<>();
+        initialProperties.put(UsernamePasswordProcessor.USERNAME.getName(), "user");
+        initialProperties.put(UsernamePasswordProcessor.PASSWORD.getName(), "#{secret-param}");
+        initialProcessor.setProperties(initialProperties);
+
+        final VersionedFlowSnapshot initialVersionSnapshot = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(initialProcessor), null);
+
+        // Update processor to have a different explicit value for both sensitive and non-sensitive properties and create a versioned flow for it.
+        final Map<String, String> updatedProperties = new HashMap<>();
+        updatedProperties.put(UsernamePasswordProcessor.USERNAME.getName(), "user");
+        updatedProperties.put(UsernamePasswordProcessor.PASSWORD.getName(), "#{other-param}");
+        initialProcessor.setProperties(updatedProperties);
+
+        final VersionedFlowSnapshot updatedVersionSnapshot = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(initialProcessor), null);
+
+        // Create child group and update to the first version of the flow, with parameter ref
+        final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
+        innerGroup.setName("Inner Group");
+        getRootGroup().addProcessGroup(innerGroup);
+
+        // Import the flow into our newly created group
+        innerGroup.updateFlow(initialVersionSnapshot, null, true, true, true);
+
+        final Set<FlowDifference> localModifications = getLocalModifications(innerGroup, updatedVersionSnapshot);
+        assertEquals(1, localModifications.size());
+        assertEquals(DifferenceType.PROPERTY_CHANGED, localModifications.iterator().next().getDifferenceType());
+    }
+
+
+    @Test
+    public void testChangeVersionFromExplicitValueToParameterSensitiveProperty() {
+        // Create a processor with a sensitive property
+        final ProcessorNode processorWithParamRef = createProcessorNode(UsernamePasswordProcessor.class);
+        processorWithParamRef.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "#{secret-param}"));
+
+        final ProcessorNode processorWithExplicitValue = createProcessorNode(UsernamePasswordProcessor.class);
+        processorWithExplicitValue.setProperties(Collections.singletonMap(UsernamePasswordProcessor.PASSWORD.getName(), "secret-value"));
+
+
+        // Create a VersionedFlowSnapshot that contains the processor
+        final Parameter parameter = new Parameter(new ParameterDescriptor.Builder().name("secret-param").sensitive(true).build(), null);
+        final VersionedFlowSnapshot versionedFlowWithParameterReference = createFlowSnapshot(Collections.emptyList(),
+            Collections.singletonList(processorWithParamRef), Collections.singleton(parameter));
+
+        final VersionedFlowSnapshot versionedFlowExplicitValue = createFlowSnapshot(Collections.emptyList(), Collections.singletonList(processorWithExplicitValue), null);
+
+        // Create child group and update to the first version of the flow, with parameter ref
+        final ProcessGroup innerGroup = getFlowController().getFlowManager().createProcessGroup("inner-group-id");
+        innerGroup.setName("Inner Group");
+        getRootGroup().addProcessGroup(innerGroup);
+
+        innerGroup.updateFlow(versionedFlowExplicitValue, null, true, true, true);
+
+        final ProcessorNode nodeInGroupWithRef = innerGroup.getProcessors().iterator().next();
+        assertNotNull(nodeInGroupWithRef.getProperty(UsernamePasswordProcessor.PASSWORD));
+
+
+        // Update the flow to new version that uses explicit value.
+        innerGroup.updateFlow(versionedFlowWithParameterReference, null, true, true, true);
+
+        // Updated flow has sensitive property that no longer references parameter. Now is an explicit value, so it should be unset
+        final ProcessorNode nodeInGroupWithNoValue = innerGroup.getProcessors().iterator().next();
+        assertEquals("#{secret-param}", nodeInGroupWithNoValue.getProperty(UsernamePasswordProcessor.PASSWORD).getRawValue());
+    }
+
+
+
+
+    private Set<FlowDifference> getLocalModifications(final ProcessGroup processGroup, final VersionedFlowSnapshot versionedFlowSnapshot) {
+        final NiFiRegistryFlowMapper mapper = new NiFiRegistryFlowMapper(getFlowController().getExtensionManager());
+        final VersionedProcessGroup localGroup = mapper.mapProcessGroup(processGroup, getFlowController().getControllerServiceProvider(), getFlowController().getFlowRegistryClient(), true);
+        final VersionedProcessGroup registryGroup = versionedFlowSnapshot.getFlowContents();
+
+        final ComparableDataFlow localFlow = new StandardComparableDataFlow("Local Flow", localGroup);
+        final ComparableDataFlow registryFlow = new StandardComparableDataFlow("Versioned Flow", registryGroup);
+
+        final Set<String> ancestorServiceIds = getAncestorGroupServiceIds(processGroup);
+        final FlowComparator flowComparator = new StandardFlowComparator(registryFlow, localFlow, ancestorServiceIds, new ConciseEvolvingDifferenceDescriptor());
+        final FlowComparison flowComparison = flowComparator.compare();
+        final Set<FlowDifference> differences = flowComparison.getDifferences().stream()
+            .filter(difference -> difference.getDifferenceType() != DifferenceType.BUNDLE_CHANGED)
+            .filter(FlowDifferenceFilters.FILTER_ADDED_REMOVED_REMOTE_PORTS)
+            .filter(FlowDifferenceFilters.FILTER_PUBLIC_PORT_NAME_CHANGES)
+            .filter(FlowDifferenceFilters.FILTER_IGNORABLE_VERSIONED_FLOW_COORDINATE_CHANGES)
+            .collect(Collectors.toCollection(HashSet::new));
+
+        return differences;
+    }
+
+    private Set<String> getAncestorGroupServiceIds(final ProcessGroup processGroup) {
+        final Set<String> ancestorServiceIds;
+        ProcessGroup parentGroup = processGroup.getParent();
+
+        if (parentGroup == null) {
+            ancestorServiceIds = Collections.emptySet();
+        } else {
+            ancestorServiceIds = parentGroup.getControllerServices(true).stream()
+                .map(cs -> {
+                    // We want to map the Controller Service to its Versioned Component ID, if it has one.
+                    // If it does not have one, we want to generate it in the same way that our Flow Mapper does
+                    // because this allows us to find the Controller Service when doing a Flow Diff.
+                    final Optional<String> versionedId = cs.getVersionedComponentId();
+                    if (versionedId.isPresent()) {
+                        return versionedId.get();
+                    }
+
+                    return UUID.nameUUIDFromBytes(cs.getIdentifier().getBytes(StandardCharsets.UTF_8)).toString();
+                })
+                .collect(Collectors.toSet());
+        }
+
+        return ancestorServiceIds;
+    }
+
+
+    private VersionedFlowSnapshot createFlowSnapshot(final List<ControllerServiceNode> controllerServices, final List<ProcessorNode> processors, final Set<Parameter> parameters) {
         final VersionedFlowSnapshotMetadata snapshotMetadata = new VersionedFlowSnapshotMetadata();
         snapshotMetadata.setAuthor("unit-test");
         snapshotMetadata.setBucketIdentifier("unit-test-bucket");
@@ -142,12 +469,14 @@ public class ImportFlowIT extends FrameworkIntegrationTest {
         for (final ProcessorNode processor : processors) {
             final VersionedProcessor versionedProcessor = flowMapper.mapProcessor(processor, getFlowController().getControllerServiceProvider(), Collections.emptySet(), new HashMap<>());
             versionedProcessors.add(versionedProcessor);
+            processor.setVersionedComponentId(versionedProcessor.getIdentifier());
         }
 
         final Set<VersionedControllerService> services = new HashSet<>();
         for (final ControllerServiceNode serviceNode : controllerServices) {
             final VersionedControllerService service = flowMapper.mapControllerService(serviceNode, getFlowController().getControllerServiceProvider(), Collections.emptySet(), new HashMap<>());
             services.add(service);
+            serviceNode.setVersionedComponentId(service.getIdentifier());
         }
 
         final VersionedProcessGroup flowContents = new VersionedProcessGroup();
@@ -164,17 +493,21 @@ public class ImportFlowIT extends FrameworkIntegrationTest {
 
         if (parameters != null) {
             final Set<VersionedParameter> versionedParameters = new HashSet<>();
-            for (final Map.Entry<String, String> entry : parameters.entrySet()) {
+            for (final Parameter parameter : parameters) {
                 final VersionedParameter versionedParameter = new VersionedParameter();
-                versionedParameter.setName(entry.getKey());
-                versionedParameter.setValue(entry.getValue());
+                versionedParameter.setName(parameter.getDescriptor().getName());
+                versionedParameter.setValue(parameter.getValue());
+                versionedParameter.setSensitive(parameter.getDescriptor().isSensitive());
+
                 versionedParameters.add(versionedParameter);
             }
 
             final VersionedParameterContext versionedParameterContext = new VersionedParameterContext();
             versionedParameterContext.setName("Unit Test Context");
             versionedParameterContext.setParameters(versionedParameters);
-            versionedFlowSnapshot.setParameterContexts(Collections.singletonMap("unit-test-context", versionedParameterContext));
+            versionedFlowSnapshot.setParameterContexts(Collections.singletonMap(versionedParameterContext.getName(), versionedParameterContext));
+
+            flowContents.setParameterContextName("Unit Test Context");
         }
 
         return versionedFlowSnapshot;
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/authorization/AuthorizeParameterReference.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/authorization/AuthorizeParameterReference.java
index fd7f644..48cbec5 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/authorization/AuthorizeParameterReference.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/authorization/AuthorizeParameterReference.java
@@ -22,14 +22,20 @@ import org.apache.nifi.components.PropertyDescriptor;
 import org.apache.nifi.groups.ProcessGroup;
 import org.apache.nifi.parameter.ExpressionLanguageAgnosticParameterParser;
 import org.apache.nifi.parameter.ParameterContext;
+import org.apache.nifi.parameter.ParameterDescriptor;
 import org.apache.nifi.parameter.ParameterParser;
 import org.apache.nifi.parameter.ParameterTokenList;
+import org.apache.nifi.registry.flow.VersionedParameter;
+import org.apache.nifi.registry.flow.VersionedParameterContext;
+import org.apache.nifi.web.NiFiServiceFacade;
 import org.apache.nifi.web.api.dto.ControllerServiceDTO;
 import org.apache.nifi.web.api.dto.FlowSnippetDTO;
 import org.apache.nifi.web.api.dto.ProcessorConfigDTO;
 import org.apache.nifi.web.api.dto.ProcessorDTO;
 
 import java.util.Map;
+import java.util.Set;
+import java.util.stream.Collectors;
 
 public class AuthorizeParameterReference {
 
@@ -124,4 +130,48 @@ public class AuthorizeParameterReference {
             }
         }
     }
+
+    /**
+     * Ensures that any Parameter Context that is referenced by the given VersionedParameterContext is readable by the given user. If the Versioned Parameter Context references a Parameter Context
+     * (by name) that does not exist in the current flow, ensures that the user has persmissions to create a new Parameter Context. If the Versioned Parameter Context contains any Parameters that
+     * do not currently exist in the Parameter Context that is referenced, ensures that the usre has permissions to WRITE to the Parameter Context so that the additional Parameter can be added.
+     *
+     * @param versionedParameterContext the Versioned Parameter Context
+     * @param serviceFacade the Service Facade
+     * @param authorizer the authorizer
+     * @param lookup the authorizable lookup
+     * @param user the user
+     */
+    public static void authorizeParameterContextAddition(final VersionedParameterContext versionedParameterContext, final NiFiServiceFacade serviceFacade, final Authorizer authorizer,
+                                                         final AuthorizableLookup lookup, final NiFiUser user) {
+        final ParameterContext parameterContext = serviceFacade.getParameterContextByName(versionedParameterContext.getName(), user);
+
+        if (parameterContext == null) {
+            // If Parameter Context does not yet exist, authorize that the user is allowed to create it.
+            lookup.getParameterContexts().authorize(authorizer, RequestAction.WRITE, user);
+            return;
+        }
+
+        // User must have READ permissions to the Parameter Context in order to use it
+        parameterContext.authorize(authorizer, RequestAction.READ, user);
+
+        // Parameter Context exists. Check if there are any new parameters that must be added.
+        final Set<String> existingParameterNames = parameterContext.getParameters().keySet().stream()
+            .map(ParameterDescriptor::getName)
+            .collect(Collectors.toSet());
+
+        boolean requiresAddition = false;
+        for (final VersionedParameter versionedParameter : versionedParameterContext.getParameters()) {
+            final String versionedParameterName = versionedParameter.getName();
+            if (!existingParameterNames.contains(versionedParameterName)) {
+                requiresAddition = true;
+                break;
+            }
+        }
+
+        if (requiresAddition) {
+            // User is required to have WRITE permission to the Parameter Context in order to add one or more parameters.
+            parameterContext.authorize(authorizer, RequestAction.WRITE, user);
+        }
+    }
 }
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/NiFiServiceFacade.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/NiFiServiceFacade.java
index 15720d8..f62bec2 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/NiFiServiceFacade.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/NiFiServiceFacade.java
@@ -25,6 +25,7 @@ import org.apache.nifi.controller.ScheduledState;
 import org.apache.nifi.controller.repository.claim.ContentDirection;
 import org.apache.nifi.controller.service.ControllerServiceState;
 import org.apache.nifi.groups.ProcessGroup;
+import org.apache.nifi.parameter.ParameterContext;
 import org.apache.nifi.registry.flow.ExternalControllerServiceReference;
 import org.apache.nifi.registry.flow.VersionedFlow;
 import org.apache.nifi.registry.flow.VersionedFlowSnapshot;
@@ -84,6 +85,7 @@ import org.apache.nifi.web.api.entity.ActivateControllerServicesEntity;
 import org.apache.nifi.web.api.entity.AffectedComponentEntity;
 import org.apache.nifi.web.api.entity.BucketEntity;
 import org.apache.nifi.web.api.entity.BulletinEntity;
+import org.apache.nifi.web.api.entity.ComponentValidationResultEntity;
 import org.apache.nifi.web.api.entity.ConnectionEntity;
 import org.apache.nifi.web.api.entity.ConnectionStatusEntity;
 import org.apache.nifi.web.api.entity.ControllerBulletinsEntity;
@@ -96,7 +98,6 @@ import org.apache.nifi.web.api.entity.FlowConfigurationEntity;
 import org.apache.nifi.web.api.entity.FlowEntity;
 import org.apache.nifi.web.api.entity.FunnelEntity;
 import org.apache.nifi.web.api.entity.LabelEntity;
-import org.apache.nifi.web.api.entity.ComponentValidationResultEntity;
 import org.apache.nifi.web.api.entity.ParameterContextEntity;
 import org.apache.nifi.web.api.entity.PortEntity;
 import org.apache.nifi.web.api.entity.PortStatusEntity;
@@ -132,6 +133,7 @@ import java.util.Map;
 import java.util.Optional;
 import java.util.Set;
 import java.util.function.Function;
+import java.util.function.Supplier;
 
 /**
  * Defines the NiFiServiceFacade interface.
@@ -1047,6 +1049,15 @@ public interface NiFiServiceFacade {
     Set<ParameterContextEntity> getParameterContexts();
 
     /**
+     * Returns the Parameter Context with the given name
+     * @param parameterContextName the name of the Parameter Context
+     * @return the Parameter Context with the given name, or <code>null</code> if no Parameter Context exists with that name
+     *
+     * @throws org.apache.nifi.authorization.AccessDeniedException if a Parameter Context exists with the given name but the user does not have READ permissions to it
+     */
+    ParameterContext getParameterContextByName(String parameterContextName, NiFiUser user);
+
+    /**
      * Returns the ParameterContextEntity for the ParameterContext with the given ID
      * @param parameterContextId the ID of the Parameter Context
      * @param user the user on whose behalf the Parameter Context is being retrieved
@@ -1547,10 +1558,11 @@ public interface NiFiServiceFacade {
      * @param updateSettings whether or not the process group's name and position should be updated
      * @param updateDescendantVersionedFlows if a child/descendant Process Group is under Version Control, specifies whether or not to
      *            update the contents of that Process Group
+     * @param  idGenerator the id generator
      * @return the Process Group
      */
     ProcessGroupEntity updateProcessGroupContents(Revision revision, String groupId, VersionControlInformationDTO versionControlInfo, VersionedFlowSnapshot snapshot,
-                                                  String componentIdSeed, boolean verifyNotModified, boolean updateSettings, boolean updateDescendantVersionedFlows);
+                                                  String componentIdSeed, boolean verifyNotModified, boolean updateSettings, boolean updateDescendantVersionedFlows, Supplier<String> idGenerator);
 
 
     /**
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java
index 7594fa8..9f4744f 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/StandardNiFiServiceFacade.java
@@ -1032,6 +1032,28 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
         return entities;
     }
 
+    @Override
+    public ParameterContext getParameterContextByName(final String parameterContextName, final NiFiUser user) {
+        final ParameterContext parameterContext = parameterContextDAO.getParameterContexts().stream()
+            .filter(context -> context.getName().equals(parameterContextName))
+            .findAny()
+            .orElse(null);
+
+        if (parameterContext == null) {
+            return null;
+        }
+
+        final boolean authorized = parameterContext.isAuthorized(authorizer, RequestAction.READ, user);
+        if (!authorized) {
+            // Note that we do not call ParameterContext.authorize() because doing so would result in an error message indicating that the user does not have permission
+            // to READ Parameter Context with ID ABC123, which tells the user that the Parameter Context ABC123 has the same name as the requested name. Instead, we simply indicate
+            // that the user is unable to read the Parameter Context and provide the name, rather than the ID, so that information about which ID corresponds to the given name is not provided.
+            throw new AccessDeniedException("Unable to read Parameter Context with name '" + parameterContextName + "'.");
+        }
+
+        return parameterContext;
+    }
+
     private ParameterContextEntity createParameterContextEntity(final ParameterContext parameterContext, final NiFiUser user) {
         final PermissionsDTO permissions = dtoFactory.createPermissionsDto(parameterContext, user);
         final RevisionDTO revisionDto = dtoFactory.createRevisionDTO(revisionManager.getRevision(parameterContext.getIdentifier()));
@@ -4854,7 +4876,8 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
 
     @Override
     public ProcessGroupEntity updateProcessGroupContents(final Revision revision, final String groupId, final VersionControlInformationDTO versionControlInfo,
-        final VersionedFlowSnapshot proposedFlowSnapshot, final String componentIdSeed, final boolean verifyNotModified, final boolean updateSettings, final boolean updateDescendantVersionedFlows) {
+                                                         final VersionedFlowSnapshot proposedFlowSnapshot, final String componentIdSeed, final boolean verifyNotModified,
+                                                         final boolean updateSettings, final boolean updateDescendantVersionedFlows, final Supplier<String> idGenerator) {
 
         final NiFiUser user = NiFiUserUtils.getNiFiUser();
 
@@ -4868,7 +4891,8 @@ public class StandardNiFiServiceFacade implements NiFiServiceFacade {
             @Override
             public RevisionUpdate<ProcessGroupDTO> update() {
                 // update the Process Group
-                processGroupDAO.updateProcessGroupFlow(groupId, proposedFlowSnapshot, versionControlInfo, componentIdSeed, verifyNotModified, updateSettings, updateDescendantVersionedFlows);
+                final ProcessGroup updatedProcessGroup = processGroupDAO.updateProcessGroupFlow(groupId, proposedFlowSnapshot, versionControlInfo, componentIdSeed, verifyNotModified, updateSettings,
+                    updateDescendantVersionedFlows);
 
                 // update the revisions
                 final Set<Revision> updatedRevisions = revisions.stream()
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ProcessGroupResource.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ProcessGroupResource.java
index 6144ab2..af2bec0 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ProcessGroupResource.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/ProcessGroupResource.java
@@ -53,6 +53,7 @@ import org.apache.nifi.registry.flow.FlowRegistryUtils;
 import org.apache.nifi.registry.flow.VersionedFlow;
 import org.apache.nifi.registry.flow.VersionedFlowSnapshot;
 import org.apache.nifi.registry.flow.VersionedFlowState;
+import org.apache.nifi.registry.flow.VersionedParameterContext;
 import org.apache.nifi.registry.variable.VariableRegistryUpdateRequest;
 import org.apache.nifi.registry.variable.VariableRegistryUpdateStep;
 import org.apache.nifi.remote.util.SiteToSiteRestApiClient;
@@ -1789,6 +1790,11 @@ public class ProcessGroupResource extends ApplicationResource {
                             final ComponentAuthorizable restrictedComponentAuthorizable = lookup.getConfigurableComponent(restrictedComponent);
                             authorizeRestrictions(authorizer, restrictedComponentAuthorizable);
                         });
+
+                        final Map<String, VersionedParameterContext> parameterContexts = versionedFlowSnapshot.getParameterContexts();
+                        if (parameterContexts != null) {
+                            parameterContexts.values().forEach(context -> AuthorizeParameterReference.authorizeParameterContextAddition(context, serviceFacade, authorizer, lookup, user));
+                        }
                     }
                 },
                 () -> {
@@ -1823,7 +1829,7 @@ public class ProcessGroupResource extends ApplicationResource {
                         // To accomplish this, we call updateProcessGroupContents() passing 'true' for the updateSettings flag but null out the position.
                         flowSnapshot.getFlowContents().setPosition(null);
                         entity = serviceFacade.updateProcessGroupContents(newGroupRevision, newGroupId, versionControlInfo, flowSnapshot,
-                                getIdGenerationSeed().orElse(null), false, true, true);
+                                getIdGenerationSeed().orElse(null), false, true, true, this::generateUuid);
                     }
 
                     populateRemainingProcessGroupEntityContent(entity);
@@ -1836,6 +1842,7 @@ public class ProcessGroupResource extends ApplicationResource {
     }
 
 
+
     private VersionedFlowSnapshot getFlowFromRegistry(final VersionControlInformationDTO versionControlInfo) {
         final VersionedFlowSnapshot flowSnapshot = serviceFacade.getVersionedFlowSnapshot(versionControlInfo, true);
         final Bucket bucket = flowSnapshot.getBucket();
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/VersionsResource.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/VersionsResource.java
index 867467f..84744e2 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/VersionsResource.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/VersionsResource.java
@@ -25,6 +25,7 @@ import io.swagger.annotations.ApiResponses;
 import io.swagger.annotations.Authorization;
 import org.apache.commons.lang3.StringUtils;
 import org.apache.nifi.authorization.AccessDeniedException;
+import org.apache.nifi.authorization.AuthorizeParameterReference;
 import org.apache.nifi.authorization.Authorizer;
 import org.apache.nifi.authorization.ComponentAuthorizable;
 import org.apache.nifi.authorization.ProcessGroupAuthorizable;
@@ -43,6 +44,7 @@ import org.apache.nifi.registry.flow.VersionedFlow;
 import org.apache.nifi.registry.flow.VersionedFlowSnapshot;
 import org.apache.nifi.registry.flow.VersionedFlowSnapshotMetadata;
 import org.apache.nifi.registry.flow.VersionedFlowState;
+import org.apache.nifi.registry.flow.VersionedParameterContext;
 import org.apache.nifi.registry.flow.VersionedProcessGroup;
 import org.apache.nifi.web.NiFiServiceFacade;
 import org.apache.nifi.web.ResourceNotFoundException;
@@ -832,7 +834,7 @@ public class VersionsResource extends ApplicationResource {
                 versionControlInfoDto.setState(flowState.name());
 
                 final ProcessGroupEntity updatedGroup = serviceFacade.updateProcessGroupContents(rev, groupId, versionControlInfoDto, flowSnapshot, getIdGenerationSeed().orElse(null), false,
-                    false, entity.getUpdateDescendantVersionedFlows());
+                    false, entity.getUpdateDescendantVersionedFlows(), this::generateUuid);
                 final VersionControlInformationDTO updatedVci = updatedGroup.getComponent().getVersionControlInformation();
 
                 final VersionControlInformationEntity responseEntity = new VersionControlInformationEntity();
@@ -1180,6 +1182,11 @@ public class VersionsResource extends ApplicationResource {
                     final ComponentAuthorizable restrictedComponentAuthorizable = lookup.getConfigurableComponent(restrictedComponent);
                     authorizeRestrictions(authorizer, restrictedComponentAuthorizable);
                 });
+
+                final Map<String, VersionedParameterContext> parameterContexts = flowSnapshot.getParameterContexts();
+                if (parameterContexts != null) {
+                    parameterContexts.values().forEach(context -> AuthorizeParameterReference.authorizeParameterContextAddition(context, serviceFacade, authorizer, lookup, user));
+                }
             },
             () -> {
                 // Step 3: Verify that all components in the snapshot exist on all nodes
@@ -1354,6 +1361,11 @@ public class VersionsResource extends ApplicationResource {
                     final ComponentAuthorizable restrictedComponentAuthorizable = lookup.getConfigurableComponent(restrictedComponent);
                     authorizeRestrictions(authorizer, restrictedComponentAuthorizable);
                 });
+
+                final Map<String, VersionedParameterContext> parameterContexts = flowSnapshot.getParameterContexts();
+                if (parameterContexts != null) {
+                    parameterContexts.values().forEach(context -> AuthorizeParameterReference.authorizeParameterContextAddition(context, serviceFacade, authorizer, lookup, user));
+                }
             },
             () -> {
                 // Step 3: Verify that all components in the snapshot exist on all nodes
@@ -1552,7 +1564,8 @@ public class VersionsResource extends ApplicationResource {
                 vci.setVersion(metadata.getVersion());
                 vci.setState(flowSnapshot.isLatest() ? VersionedFlowState.UP_TO_DATE.name() : VersionedFlowState.STALE.name());
 
-                serviceFacade.updateProcessGroupContents(revision, groupId, vci, flowSnapshot, idGenerationSeed, verifyNotModified, false, updateDescendantVersionedFlows);
+                serviceFacade.updateProcessGroupContents(revision, groupId, vci, flowSnapshot, idGenerationSeed, verifyNotModified, false, updateDescendantVersionedFlows,
+                    this::generateUuid);
             }
         } finally {
             if (!asyncRequest.isCancelled()) {
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/dto/DtoFactory.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/dto/DtoFactory.java
index 4211707..97b8c5e 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/dto/DtoFactory.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-web/nifi-web-api/src/main/java/org/apache/nifi/web/api/dto/DtoFactory.java
@@ -1405,7 +1405,9 @@ public final class DtoFactory {
         dto.setName(descriptor.getName());
         dto.setDescription(descriptor.getDescription());
         dto.setSensitive(descriptor.isSensitive());
-        dto.setValue(descriptor.isSensitive() ? SENSITIVE_VALUE_MASK : parameter.getValue());
+        if (parameter.getValue() != null) {
+            dto.setValue(descriptor.isSensitive() ? SENSITIVE_VALUE_MASK : parameter.getValue());
+        }
 
         final ParameterReferenceManager parameterReferenceManager = parameterContext.getParameterReferenceManager();