nifi-commits mailing list archives

Site index · List index
Message view « Date » · « Thread »
Top « Date » · « Thread »
From alopre...@apache.org
Subject [12/13] nifi git commit: NIFI-3594 Implemented encrypted provenance repository. Added src/test/resources/logback-test.xml files resetting log level from DEBUG (in nifi-data-provenance-utils) to WARN because later tests depend on MockComponentLog recordin
Date Tue, 02 May 2017 17:27:37 GMT
http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-data-provenance-utils/src/test/groovy/org/apache/nifi/provenance/CryptoUtilsTest.groovy
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-data-provenance-utils/src/test/groovy/org/apache/nifi/provenance/CryptoUtilsTest.groovy b/nifi-commons/nifi-data-provenance-utils/src/test/groovy/org/apache/nifi/provenance/CryptoUtilsTest.groovy
new file mode 100644
index 0000000..162896f
--- /dev/null
+++ b/nifi-commons/nifi-data-provenance-utils/src/test/groovy/org/apache/nifi/provenance/CryptoUtilsTest.groovy
@@ -0,0 +1,436 @@
+/*
+ * 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.provenance
+
+import org.bouncycastle.jce.provider.BouncyCastleProvider
+import org.bouncycastle.util.encoders.Hex
+import org.junit.After
+import org.junit.AfterClass
+import org.junit.Before
+import org.junit.BeforeClass
+import org.junit.ClassRule
+import org.junit.Test
+import org.junit.rules.TemporaryFolder
+import org.junit.runner.RunWith
+import org.junit.runners.JUnit4
+import org.slf4j.Logger
+import org.slf4j.LoggerFactory
+
+import javax.crypto.Cipher
+import javax.crypto.SecretKey
+import javax.crypto.spec.IvParameterSpec
+import javax.crypto.spec.SecretKeySpec
+import java.nio.charset.StandardCharsets
+import java.nio.file.Files
+import java.nio.file.attribute.PosixFilePermission
+import java.security.KeyManagementException
+import java.security.SecureRandom
+import java.security.Security
+
+import static groovy.test.GroovyAssert.shouldFail
+
+@RunWith(JUnit4.class)
+class CryptoUtilsTest {
+    private static final Logger logger = LoggerFactory.getLogger(CryptoUtilsTest.class)
+
+    private static final String KEY_ID = "K1"
+    private static final String KEY_HEX_128 = "0123456789ABCDEFFEDCBA9876543210"
+    private static final String KEY_HEX_256 = KEY_HEX_128 * 2
+    private static final String KEY_HEX = isUnlimitedStrengthCryptoAvailable() ? KEY_HEX_256 : KEY_HEX_128
+
+    private static
+    final Set<PosixFilePermission> ALL_POSIX_ATTRS = PosixFilePermission.values() as Set<PosixFilePermission>
+
+    @ClassRule
+    public static TemporaryFolder tempFolder = new TemporaryFolder()
+
+    @BeforeClass
+    static void setUpOnce() throws Exception {
+        Security.addProvider(new BouncyCastleProvider())
+
+        logger.metaClass.methodMissing = { String name, args ->
+            logger.info("[${name?.toUpperCase()}] ${(args as List).join(" ")}")
+        }
+    }
+
+    @Before
+    void setUp() throws Exception {
+        tempFolder.create()
+    }
+
+    @After
+    void tearDown() throws Exception {
+        tempFolder?.delete()
+    }
+
+    @AfterClass
+    static void tearDownOnce() throws Exception {
+
+    }
+
+    private static boolean isUnlimitedStrengthCryptoAvailable() {
+        Cipher.getMaxAllowedKeyLength("AES") > 128
+    }
+
+    @Test
+    void testShouldConcatenateByteArrays() {
+        // Arrange
+        byte[] bytes1 = "These are some bytes".getBytes(StandardCharsets.UTF_8)
+        byte[] bytes2 = "These are some other bytes".getBytes(StandardCharsets.UTF_8)
+        final byte[] EXPECTED_CONCATENATED_BYTES = ((bytes1 as List) << (bytes2 as List)).flatten() as byte[]
+        logger.info("Expected concatenated bytes: ${Hex.toHexString(EXPECTED_CONCATENATED_BYTES)}")
+
+        // Act
+        byte[] concat = CryptoUtils.concatByteArrays(bytes1, bytes2)
+        logger.info("  Actual concatenated bytes: ${Hex.toHexString(concat)}")
+
+        // Assert
+        assert concat == EXPECTED_CONCATENATED_BYTES
+    }
+
+    @Test
+    void testShouldValidateStaticKeyProvider() {
+        // Arrange
+        String staticProvider = StaticKeyProvider.class.name
+        String providerLocation = null
+
+        // Act
+        boolean keyProviderIsValid = CryptoUtils.isValidKeyProvider(staticProvider, providerLocation, KEY_ID, [(KEY_ID): KEY_HEX])
+        logger.info("Key Provider ${staticProvider} with location ${providerLocation} and keyId ${KEY_ID} / ${KEY_HEX} is ${keyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert keyProviderIsValid
+    }
+
+    @Test
+    void testShouldNotValidateStaticKeyProviderMissingKeyId() {
+        // Arrange
+        String staticProvider = StaticKeyProvider.class.name
+        String providerLocation = null
+
+        // Act
+        boolean keyProviderIsValid = CryptoUtils.isValidKeyProvider(staticProvider, providerLocation, null, [(KEY_ID): KEY_HEX])
+        logger.info("Key Provider ${staticProvider} with location ${providerLocation} and keyId ${null} / ${KEY_HEX} is ${keyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert !keyProviderIsValid
+    }
+
+    @Test
+    void testShouldNotValidateStaticKeyProviderMissingKey() {
+        // Arrange
+        String staticProvider = StaticKeyProvider.class.name
+        String providerLocation = null
+
+        // Act
+        boolean keyProviderIsValid = CryptoUtils.isValidKeyProvider(staticProvider, providerLocation, KEY_ID, null)
+        logger.info("Key Provider ${staticProvider} with location ${providerLocation} and keyId ${KEY_ID} / ${null} is ${keyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert !keyProviderIsValid
+    }
+
+    @Test
+    void testShouldNotValidateStaticKeyProviderWithInvalidKey() {
+        // Arrange
+        String staticProvider = StaticKeyProvider.class.name
+        String providerLocation = null
+
+        // Act
+        boolean keyProviderIsValid = CryptoUtils.isValidKeyProvider(staticProvider, providerLocation, KEY_ID, [(KEY_ID): KEY_HEX[0..<-2]])
+        logger.info("Key Provider ${staticProvider} with location ${providerLocation} and keyId ${KEY_ID} / ${KEY_HEX[0..<-2]} is ${keyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert !keyProviderIsValid
+    }
+
+    @Test
+    void testShouldValidateFileBasedKeyProvider() {
+        // Arrange
+        String fileBasedProvider = FileBasedKeyProvider.class.name
+        File fileBasedProviderFile = tempFolder.newFile("filebased.kp")
+        String providerLocation = fileBasedProviderFile.path
+        logger.info("Created temporary file based key provider: ${providerLocation}")
+
+        // Act
+        boolean keyProviderIsValid = CryptoUtils.isValidKeyProvider(fileBasedProvider, providerLocation, KEY_ID, null)
+        logger.info("Key Provider ${fileBasedProvider} with location ${providerLocation} and keyId ${KEY_ID} / ${null} is ${keyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert keyProviderIsValid
+    }
+
+    @Test
+    void testShouldNotValidateUnreadableOrMissingFileBasedKeyProvider() {
+        // Arrange
+        String fileBasedProvider = FileBasedKeyProvider.class.name
+        File fileBasedProviderFile = tempFolder.newFile("filebased.kp")
+        String providerLocation = fileBasedProviderFile.path
+        logger.info("Created temporary file based key provider: ${providerLocation}")
+
+        // Make it unreadable
+        fileBasedProviderFile.setReadable(false, false)
+        Files.setPosixFilePermissions(fileBasedProviderFile.toPath(), [] as Set<PosixFilePermission>)
+
+        // Act
+        boolean unreadableKeyProviderIsValid = CryptoUtils.isValidKeyProvider(fileBasedProvider, providerLocation, KEY_ID, null)
+        logger.info("Key Provider ${fileBasedProvider} with location ${providerLocation} and keyId ${KEY_ID} / ${null} is ${unreadableKeyProviderIsValid ? "valid" : "invalid"}")
+
+        String missingLocation = providerLocation + "_missing"
+        boolean missingKeyProviderIsValid = CryptoUtils.isValidKeyProvider(fileBasedProvider, missingLocation, KEY_ID, null)
+        logger.info("Key Provider ${fileBasedProvider} with location ${missingLocation} and keyId ${KEY_ID} / ${null} is ${missingKeyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert !unreadableKeyProviderIsValid
+        assert !missingKeyProviderIsValid
+
+        // Make the file deletable so cleanup can occur
+        fileBasedProviderFile.setReadable(true, false)
+        Files.setPosixFilePermissions(fileBasedProviderFile.toPath(), ALL_POSIX_ATTRS)
+    }
+
+    @Test
+    void testShouldNotValidateFileBasedKeyProviderMissingKeyId() {
+        // Arrange
+        String fileBasedProvider = FileBasedKeyProvider.class.name
+        File fileBasedProviderFile = tempFolder.newFile("missing_key_id.kp")
+        String providerLocation = fileBasedProviderFile.path
+        logger.info("Created temporary file based key provider: ${providerLocation}")
+
+        // Act
+        boolean keyProviderIsValid = CryptoUtils.isValidKeyProvider(fileBasedProvider, providerLocation, null, null)
+        logger.info("Key Provider ${fileBasedProvider} with location ${providerLocation} and keyId ${null} / ${null} is ${keyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert !keyProviderIsValid
+    }
+
+    @Test
+    void testShouldNotValidateUnknownKeyProvider() {
+        // Arrange
+        String providerImplementation = "org.apache.nifi.provenance.ImaginaryKeyProvider"
+        String providerLocation = null
+
+        // Act
+        boolean keyProviderIsValid = CryptoUtils.isValidKeyProvider(providerImplementation, providerLocation, KEY_ID, null)
+        logger.info("Key Provider ${providerImplementation} with location ${providerLocation} and keyId ${KEY_ID} / ${null} is ${keyProviderIsValid ? "valid" : "invalid"}")
+
+        // Assert
+        assert !keyProviderIsValid
+    }
+
+    @Test
+    void testShouldValidateKey() {
+        // Arrange
+        String validKey = KEY_HEX
+        String validLowercaseKey = KEY_HEX.toLowerCase()
+
+        String tooShortKey = KEY_HEX[0..<-2]
+        String tooLongKey = KEY_HEX + KEY_HEX // Guaranteed to be 2x the max valid key length
+        String nonHexKey = KEY_HEX.replaceFirst(/A/, "X")
+
+        def validKeys = [validKey, validLowercaseKey]
+        def invalidKeys = [tooShortKey, tooLongKey, nonHexKey]
+
+        // If unlimited strength is available, also validate 128 and 196 bit keys
+        if (isUnlimitedStrengthCryptoAvailable()) {
+            validKeys << KEY_HEX_128
+            validKeys << KEY_HEX_256[0..<48]
+        } else {
+            invalidKeys << KEY_HEX_256[0..<48]
+            invalidKeys << KEY_HEX_256
+        }
+
+        // Act
+        def validResults = validKeys.collect { String key ->
+            logger.info("Validating ${key}")
+            CryptoUtils.keyIsValid(key)
+        }
+
+        def invalidResults = invalidKeys.collect { String key ->
+            logger.info("Validating ${key}")
+            CryptoUtils.keyIsValid(key)
+        }
+
+        // Assert
+        assert validResults.every()
+        assert invalidResults.every { !it }
+    }
+
+    @Test
+    void testShouldReadKeys() {
+        // Arrange
+        String masterKeyHex = KEY_HEX
+        SecretKey masterKey = new SecretKeySpec(Hex.decode(masterKeyHex), "AES")
+
+        // Generate the file
+        String keyFileName = "keys.nkp"
+        File keyFile = tempFolder.newFile(keyFileName)
+        final int KEY_COUNT = 5
+        List<String> lines = []
+        KEY_COUNT.times { int i ->
+            lines.add("key${i + 1}=${generateEncryptedKey(masterKey)}")
+        }
+
+        keyFile.text = lines.join("\n")
+
+        logger.info("File contents: \n${keyFile.text}")
+
+        // Act
+        def readKeys = CryptoUtils.readKeys(keyFile.path, masterKey)
+        logger.info("Read ${readKeys.size()} keys from ${keyFile.path}")
+
+        // Assert
+        assert readKeys.size() == KEY_COUNT
+    }
+
+    @Test
+    void testShouldReadKeysWithDuplicates() {
+        // Arrange
+        String masterKeyHex = KEY_HEX
+        SecretKey masterKey = new SecretKeySpec(Hex.decode(masterKeyHex), "AES")
+
+        // Generate the file
+        String keyFileName = "keys.nkp"
+        File keyFile = tempFolder.newFile(keyFileName)
+        final int KEY_COUNT = 3
+        List<String> lines = []
+        KEY_COUNT.times { int i ->
+            lines.add("key${i + 1}=${generateEncryptedKey(masterKey)}")
+        }
+
+        lines.add("key3=${generateEncryptedKey(masterKey)}")
+
+        keyFile.text = lines.join("\n")
+
+        logger.info("File contents: \n${keyFile.text}")
+
+        // Act
+        def readKeys = CryptoUtils.readKeys(keyFile.path, masterKey)
+        logger.info("Read ${readKeys.size()} keys from ${keyFile.path}")
+
+        // Assert
+        assert readKeys.size() == KEY_COUNT
+    }
+
+    @Test
+    void testShouldReadKeysWithSomeMalformed() {
+        // Arrange
+        String masterKeyHex = KEY_HEX
+        SecretKey masterKey = new SecretKeySpec(Hex.decode(masterKeyHex), "AES")
+
+        // Generate the file
+        String keyFileName = "keys.nkp"
+        File keyFile = tempFolder.newFile(keyFileName)
+        final int KEY_COUNT = 5
+        List<String> lines = []
+        KEY_COUNT.times { int i ->
+            lines.add("key${i + 1}=${generateEncryptedKey(masterKey)}")
+        }
+
+        // Insert the malformed keys in the middle
+        lines.add(2, "keyX1==${generateEncryptedKey(masterKey)}")
+        lines.add(4, "=${generateEncryptedKey(masterKey)}")
+        lines.add(6, "keyX3=non Base64-encoded data")
+
+        keyFile.text = lines.join("\n")
+
+        logger.info("File contents: \n${keyFile.text}")
+
+        // Act
+        def readKeys = CryptoUtils.readKeys(keyFile.path, masterKey)
+        logger.info("Read ${readKeys.size()} keys from ${keyFile.path}")
+
+        // Assert
+        assert readKeys.size() == KEY_COUNT
+    }
+
+    @Test
+    void testShouldNotReadKeysIfAllMalformed() {
+        // Arrange
+        String masterKeyHex = KEY_HEX
+        SecretKey masterKey = new SecretKeySpec(Hex.decode(masterKeyHex), "AES")
+
+        // Generate the file
+        String keyFileName = "keys.nkp"
+        File keyFile = tempFolder.newFile(keyFileName)
+        final int KEY_COUNT = 5
+        List<String> lines = []
+
+        // All of these keys are malformed
+        KEY_COUNT.times { int i ->
+            lines.add("key${i + 1}=${generateEncryptedKey(masterKey)[0..<-4]}")
+        }
+
+        keyFile.text = lines.join("\n")
+
+        logger.info("File contents: \n${keyFile.text}")
+
+        // Act
+        def msg = shouldFail(KeyManagementException) {
+            def readKeys = CryptoUtils.readKeys(keyFile.path, masterKey)
+            logger.info("Read ${readKeys.size()} keys from ${keyFile.path}")
+        }
+
+        // Assert
+        assert msg.getMessage() == "The provided file contained no valid keys"
+    }
+
+    @Test
+    void testShouldNotReadKeysIfEmptyOrMissing() {
+        // Arrange
+        String masterKeyHex = KEY_HEX
+        SecretKey masterKey = new SecretKeySpec(Hex.decode(masterKeyHex), "AES")
+
+        // Generate the file
+        String keyFileName = "empty.nkp"
+        File keyFile = tempFolder.newFile(keyFileName)
+        logger.info("File contents: \n${keyFile.text}")
+
+        // Act
+        def missingMsg = shouldFail(KeyManagementException) {
+            def readKeys = CryptoUtils.readKeys(keyFile.path, masterKey)
+            logger.info("Read ${readKeys.size()} keys from ${keyFile.path}")
+        }
+        logger.expected("Missing file: ${missingMsg}")
+
+        def emptyMsg = shouldFail(KeyManagementException) {
+            def readKeys = CryptoUtils.readKeys(null, masterKey)
+            logger.info("Read ${readKeys.size()} keys from ${null}")
+        }
+        logger.expected("Empty file: ${emptyMsg}")
+
+        // Assert
+        assert missingMsg.getMessage() == "The provided file contained no valid keys"
+        assert emptyMsg.getMessage() == "The key provider file is not present and readable"
+    }
+
+    private static String generateEncryptedKey(SecretKey masterKey) {
+        byte[] ivBytes = new byte[16]
+        byte[] keyBytes = new byte[isUnlimitedStrengthCryptoAvailable() ? 32 : 16]
+
+        SecureRandom sr = new SecureRandom()
+        sr.nextBytes(ivBytes)
+        sr.nextBytes(keyBytes)
+
+        Cipher masterCipher = Cipher.getInstance("AES/GCM/NoPadding", "BC")
+        masterCipher.init(Cipher.ENCRYPT_MODE, masterKey, new IvParameterSpec(ivBytes))
+        byte[] cipherBytes = masterCipher.doFinal(keyBytes)
+
+        Base64.encoder.encodeToString(CryptoUtils.concatByteArrays(ivBytes, cipherBytes))
+    }
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-data-provenance-utils/src/test/java/org/apache/nifi/provenance/EncryptionExceptionTest.java
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-data-provenance-utils/src/test/java/org/apache/nifi/provenance/EncryptionExceptionTest.java b/nifi-commons/nifi-data-provenance-utils/src/test/java/org/apache/nifi/provenance/EncryptionExceptionTest.java
new file mode 100644
index 0000000..e23d492
--- /dev/null
+++ b/nifi-commons/nifi-data-provenance-utils/src/test/java/org/apache/nifi/provenance/EncryptionExceptionTest.java
@@ -0,0 +1,27 @@
+/*
+ * 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.provenance;
+
+import org.junit.Test;
+
+public class EncryptionExceptionTest {
+
+    @Test
+    public void testShouldTriggerGroovyTestExecution() {
+        // This method does nothing but tell Maven to run the groovy tests
+    }
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-data-provenance-utils/src/test/resources/logback-test.xml
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-data-provenance-utils/src/test/resources/logback-test.xml b/nifi-commons/nifi-data-provenance-utils/src/test/resources/logback-test.xml
new file mode 100644
index 0000000..c10508d
--- /dev/null
+++ b/nifi-commons/nifi-data-provenance-utils/src/test/resources/logback-test.xml
@@ -0,0 +1,32 @@
+<?xml version="1.0" encoding="UTF-8"?>
+<!--
+  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.
+-->
+
+<configuration scan="true" scanPeriod="30 seconds">
+    <appender name="CONSOLE" class="ch.qos.logback.core.ConsoleAppender">
+        <encoder class="ch.qos.logback.classic.encoder.PatternLayoutEncoder">
+            <pattern>%-4r [%t] %-5p %c - %m%n</pattern>
+        </encoder>
+    </appender>
+
+    <!-- valid logging levels: TRACE, DEBUG, INFO, WARN, ERROR -->
+    <logger name="org.apache.nifi" level="DEBUG"/>
+
+    <root level="INFO">
+        <appender-ref ref="CONSOLE"/>
+    </root>
+
+</configuration>
+

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-properties/src/main/java/org/apache/nifi/util/NiFiProperties.java
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-properties/src/main/java/org/apache/nifi/util/NiFiProperties.java b/nifi-commons/nifi-properties/src/main/java/org/apache/nifi/util/NiFiProperties.java
index 3d657fb..d69a280 100644
--- a/nifi-commons/nifi-properties/src/main/java/org/apache/nifi/util/NiFiProperties.java
+++ b/nifi-commons/nifi-properties/src/main/java/org/apache/nifi/util/NiFiProperties.java
@@ -32,6 +32,7 @@ import java.util.List;
 import java.util.Map;
 import java.util.Properties;
 import java.util.Set;
+import java.util.stream.Collectors;
 
 /**
  * The NiFiProperties class holds all properties which are needed for various
@@ -116,6 +117,11 @@ public abstract class NiFiProperties {
     public static final String PROVENANCE_INDEXED_ATTRIBUTES = "nifi.provenance.repository.indexed.attributes";
     public static final String PROVENANCE_INDEX_SHARD_SIZE = "nifi.provenance.repository.index.shard.size";
     public static final String PROVENANCE_JOURNAL_COUNT = "nifi.provenance.repository.journal.count";
+    public static final String PROVENANCE_REPO_ENCRYPTION_KEY = "nifi.provenance.repository.encryption.key";
+    public static final String PROVENANCE_REPO_ENCRYPTION_KEY_ID = "nifi.provenance.repository.encryption.key.id";
+    public static final String PROVENANCE_REPO_ENCRYPTION_KEY_PROVIDER_IMPLEMENTATION_CLASS = "nifi.provenance.repository.encryption.key.provider.implementation";
+    public static final String PROVENANCE_REPO_ENCRYPTION_KEY_PROVIDER_LOCATION = "nifi.provenance.repository.encryption.key.provider.location";
+    public static final String PROVENANCE_REPO_DEBUG_FREQUENCY = "nifi.provenance.repository.debug.frequency";
 
     // component status repository properties
     public static final String COMPONENT_STATUS_REPOSITORY_IMPLEMENTATION = "nifi.components.status.repository.implementation";
@@ -769,7 +775,7 @@ public abstract class NiFiProperties {
     /**
      * Returns true if client certificates are required for REST API. Determined
      * if the following conditions are all true:
-     *
+     * <p>
      * - login identity provider is not populated
      * - Kerberos service support is not enabled
      *
@@ -1034,6 +1040,61 @@ public abstract class NiFiProperties {
         return getPropertyKeys().size();
     }
 
+    public String getProvenanceRepoEncryptionKeyId() {
+        return getProperty(PROVENANCE_REPO_ENCRYPTION_KEY_ID);
+    }
+
+    /**
+     * Returns the active provenance repository encryption key if a {@code StaticKeyProvider} is in use.
+     * If no key ID is specified in the properties file, the default
+     * {@code nifi.provenance.repository.encryption.key} value is returned. If a key ID is specified in
+     * {@code nifi.provenance.repository.encryption.key.id}, it will attempt to read from
+     * {@code nifi.provenance.repository.encryption.key.id.XYZ} where {@code XYZ} is the provided key
+     * ID. If that value is empty, it will use the default property
+     * {@code nifi.provenance.repository.encryption.key}.
+     *
+     * @return the provenance repository encryption key in hex form
+     */
+    public String getProvenanceRepoEncryptionKey() {
+        String keyId = getProvenanceRepoEncryptionKeyId();
+        String keyKey = StringUtils.isBlank(keyId) ? PROVENANCE_REPO_ENCRYPTION_KEY : PROVENANCE_REPO_ENCRYPTION_KEY + ".id." + keyId;
+        return getProperty(keyKey, getProperty(PROVENANCE_REPO_ENCRYPTION_KEY));
+    }
+
+    /**
+     * Returns a map of keyId -> key in hex loaded from the {@code nifi.properties} file if a
+     * {@code StaticKeyProvider} is defined. If {@code FileBasedKeyProvider} is defined, use
+     * {@code CryptoUtils#readKeys()} instead -- this method will return an empty map.
+     *
+     * @return a Map of the keys identified by key ID
+     */
+    public Map<String, String> getProvenanceRepoEncryptionKeys() {
+        Map<String, String> keys = new HashMap<>();
+        List<String> keyProperties = getProvenanceRepositoryEncryptionKeyProperties();
+
+        // Retrieve the actual key values and store non-empty values in the map
+        for (String prop : keyProperties) {
+            final String value = getProperty(prop);
+            if (!StringUtils.isBlank(value)) {
+                if (prop.equalsIgnoreCase(PROVENANCE_REPO_ENCRYPTION_KEY)) {
+                    prop = getProvenanceRepoEncryptionKeyId();
+                } else {
+                    // Extract nifi.provenance.repository.encryption.key.id.key1 -> key1
+                    prop = prop.substring(prop.lastIndexOf(".") + 1);
+                }
+                keys.put(prop, value);
+            }
+        }
+        return keys;
+    }
+
+    private List<String> getProvenanceRepositoryEncryptionKeyProperties() {
+        // Filter all the property keys that define a key
+        return getPropertyKeys().stream().filter(k ->
+                k.startsWith(PROVENANCE_REPO_ENCRYPTION_KEY_ID + ".") || k.equalsIgnoreCase(PROVENANCE_REPO_ENCRYPTION_KEY)
+        ).collect(Collectors.toList());
+    }
+
     /**
      * Creates an instance of NiFiProperties. This should likely not be called
      * by any classes outside of the NiFi framework but can be useful by the
@@ -1042,11 +1103,11 @@ public abstract class NiFiProperties {
      * file specified cannot be found/read a runtime exception will be thrown.
      * If one is not specified no properties will be loaded by default.
      *
-     * @param propertiesFilePath if provided properties will be loaded from
-     * given file; else will be loaded from System property. Can be null.
+     * @param propertiesFilePath   if provided properties will be loaded from
+     *                             given file; else will be loaded from System property. Can be null.
      * @param additionalProperties allows overriding of properties with the
-     * supplied values. these will be applied after loading from any properties
-     * file. Can be null or empty.
+     *                             supplied values. these will be applied after loading from any properties
+     *                             file. Can be null or empty.
      * @return NiFiProperties
      */
     public static NiFiProperties createBasicNiFiProperties(final String propertiesFilePath, final Map<String, String> additionalProperties) {
@@ -1108,10 +1169,9 @@ public abstract class NiFiProperties {
     public void validate() {
         // REMOTE_INPUT_HOST should be a valid hostname
         String remoteInputHost = getProperty(REMOTE_INPUT_HOST);
-        if(!StringUtils.isBlank(remoteInputHost) && remoteInputHost.split(":").length > 1) { // no scheme/port needed here (http://)
+        if (!StringUtils.isBlank(remoteInputHost) && remoteInputHost.split(":").length > 1) { // no scheme/port needed here (http://)
             throw new IllegalArgumentException(remoteInputHost + " is not a correct value for " + REMOTE_INPUT_HOST + ". It should be a valid hostname without protocol or port.");
         }
         // Other properties to validate...
     }
-
 }

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/pom.xml
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/pom.xml b/nifi-commons/nifi-security-utils/pom.xml
index e2e5ee1..b004c8b 100644
--- a/nifi-commons/nifi-security-utils/pom.xml
+++ b/nifi-commons/nifi-security-utils/pom.xml
@@ -44,6 +44,10 @@
             <artifactId>commons-lang3</artifactId>
         </dependency>
         <dependency>
+            <groupId>commons-codec</groupId>
+            <artifactId>commons-codec</artifactId>
+        </dependency>
+        <dependency>
             <groupId>org.bouncycastle</groupId>
             <artifactId>bcprov-jdk15on</artifactId>
         </dependency>

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/EncryptionMethod.java
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/EncryptionMethod.java b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/EncryptionMethod.java
index 53664f1..a1ef2a4 100644
--- a/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/EncryptionMethod.java
+++ b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/EncryptionMethod.java
@@ -16,13 +16,13 @@
  */
 package org.apache.nifi.security.util;
 
+import org.apache.commons.lang3.StringUtils;
 import org.apache.commons.lang3.builder.ToStringBuilder;
 import org.apache.commons.lang3.builder.ToStringStyle;
 
 /**
  * Enumeration capturing essential information about the various encryption
  * methods that might be supported.
- *
  */
 public enum EncryptionMethod {
 
@@ -105,4 +105,15 @@ public enum EncryptionMethod {
         builder.append("Keyed cipher", isKeyedCipher());
         return builder.toString();
     }
+
+    public static EncryptionMethod forAlgorithm(String algorithm) {
+        if (StringUtils.isNotBlank(algorithm)) {
+            for (EncryptionMethod em : EncryptionMethod.values()) {
+                if (em.algorithm.equalsIgnoreCase(algorithm)) {
+                    return em;
+                }
+            }
+        }
+        return null;
+    }
 }

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/AESKeyedCipherProvider.java
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/AESKeyedCipherProvider.java b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/AESKeyedCipherProvider.java
new file mode 100644
index 0000000..617a3e5
--- /dev/null
+++ b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/AESKeyedCipherProvider.java
@@ -0,0 +1,152 @@
+/*
+ * 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.security.util.crypto;
+
+import java.io.UnsupportedEncodingException;
+import java.security.InvalidAlgorithmParameterException;
+import java.security.InvalidKeyException;
+import java.security.NoSuchAlgorithmException;
+import java.security.NoSuchProviderException;
+import java.security.SecureRandom;
+import java.security.spec.InvalidKeySpecException;
+import java.util.Arrays;
+import java.util.List;
+import javax.crypto.Cipher;
+import javax.crypto.NoSuchPaddingException;
+import javax.crypto.SecretKey;
+import javax.crypto.spec.IvParameterSpec;
+import org.apache.commons.lang3.StringUtils;
+import org.apache.nifi.processor.exception.ProcessException;
+import org.apache.nifi.security.util.EncryptionMethod;
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+/**
+ * This is a standard implementation of {@link KeyedCipherProvider} which supports {@code AES} cipher families with arbitrary modes of operation (currently only {@code CBC}, {@code CTR}, and {@code
+ * GCM} are supported as {@link EncryptionMethod}s.
+ */
+public class AESKeyedCipherProvider extends KeyedCipherProvider {
+    private static final Logger logger = LoggerFactory.getLogger(AESKeyedCipherProvider.class);
+    private static final int IV_LENGTH = 16;
+    private static final List<Integer> VALID_KEY_LENGTHS = Arrays.asList(128, 192, 256);
+
+    /**
+     * Returns an initialized cipher for the specified algorithm. The IV is provided externally to allow for non-deterministic IVs, as IVs
+     * deterministically derived from the password are a potential vulnerability and compromise semantic security. See
+     * <a href="http://crypto.stackexchange.com/a/3970/12569">Ilmari Karonen's answer on Crypto Stack Exchange</a>
+     *
+     * @param encryptionMethod the {@link EncryptionMethod}
+     * @param key              the key
+     * @param iv               the IV or nonce (cannot be all 0x00)
+     * @param encryptMode      true for encrypt, false for decrypt
+     * @return the initialized cipher
+     * @throws Exception if there is a problem initializing the cipher
+     */
+    @Override
+    public Cipher getCipher(EncryptionMethod encryptionMethod, SecretKey key, byte[] iv, boolean encryptMode) throws Exception {
+        try {
+            return getInitializedCipher(encryptionMethod, key, iv, encryptMode);
+        } catch (IllegalArgumentException e) {
+            throw e;
+        } catch (Exception e) {
+            throw new ProcessException("Error initializing the cipher", e);
+        }
+    }
+
+    /**
+     * Returns an initialized cipher for the specified algorithm. The IV will be generated internally (for encryption). If decryption is requested, it will throw an exception.
+     *
+     * @param encryptionMethod the {@link EncryptionMethod}
+     * @param key              the key
+     * @param encryptMode      true for encrypt, false for decrypt
+     * @return the initialized cipher
+     * @throws Exception if there is a problem initializing the cipher or if decryption is requested
+     */
+    @Override
+    public Cipher getCipher(EncryptionMethod encryptionMethod, SecretKey key, boolean encryptMode) throws Exception {
+        return getCipher(encryptionMethod, key, new byte[0], encryptMode);
+    }
+
+    protected Cipher getInitializedCipher(EncryptionMethod encryptionMethod, SecretKey key, byte[] iv,
+                                          boolean encryptMode) throws NoSuchAlgorithmException, NoSuchProviderException,
+            InvalidKeySpecException, NoSuchPaddingException, InvalidKeyException, InvalidAlgorithmParameterException, UnsupportedEncodingException {
+        if (encryptionMethod == null) {
+            throw new IllegalArgumentException("The encryption method must be specified");
+        }
+
+        if (!encryptionMethod.isKeyedCipher()) {
+            throw new IllegalArgumentException(encryptionMethod.name() + " requires a PBECipherProvider");
+        }
+
+        String algorithm = encryptionMethod.getAlgorithm();
+        String provider = encryptionMethod.getProvider();
+
+        if (key == null) {
+            throw new IllegalArgumentException("The key must be specified");
+        }
+
+        if (!isValidKeyLength(key)) {
+            throw new IllegalArgumentException("The key must be of length [" + StringUtils.join(VALID_KEY_LENGTHS, ", ") + "]");
+        }
+
+        Cipher cipher = Cipher.getInstance(algorithm, provider);
+        final String operation = encryptMode ? "encrypt" : "decrypt";
+
+        boolean ivIsInvalid = false;
+
+        // If an IV was not provided already, generate a random IV and inject it in the cipher
+        int ivLength = cipher.getBlockSize();
+        if (iv.length != ivLength) {
+            logger.warn("An IV was provided of length {} bytes for {}ion but should be {} bytes", iv.length, operation, ivLength);
+            ivIsInvalid = true;
+        }
+
+        final byte[] emptyIv = new byte[ivLength];
+        if (Arrays.equals(iv, emptyIv)) {
+            logger.warn("An empty IV was provided of length {} for {}ion", iv.length, operation);
+            ivIsInvalid = true;
+        }
+
+        if (ivIsInvalid) {
+            if (encryptMode) {
+                logger.warn("Generating new IV. The value can be obtained in the calling code by invoking 'cipher.getIV()';");
+                iv = generateIV();
+            } else {
+                // Can't decrypt without an IV
+                throw new IllegalArgumentException("Cannot decrypt without a valid IV");
+            }
+        }
+        cipher.init(encryptMode ? Cipher.ENCRYPT_MODE : Cipher.DECRYPT_MODE, key, new IvParameterSpec(iv));
+
+        return cipher;
+    }
+
+    private boolean isValidKeyLength(SecretKey key) {
+        return VALID_KEY_LENGTHS.contains(key.getEncoded().length * 8);
+    }
+
+    /**
+     * Generates a new random IV of 16 bytes using {@link java.security.SecureRandom}.
+     *
+     * @return the IV
+     */
+    public byte[] generateIV() {
+        byte[] iv = new byte[IV_LENGTH];
+        new SecureRandom().nextBytes(iv);
+        return iv;
+    }
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherProvider.java
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherProvider.java b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherProvider.java
new file mode 100644
index 0000000..e3632b2
--- /dev/null
+++ b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherProvider.java
@@ -0,0 +1,23 @@
+/*
+ * 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.security.util.crypto;
+
+/**
+ * Marker interface for cipher providers.
+ */
+public interface CipherProvider {
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherUtility.java
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherUtility.java b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherUtility.java
new file mode 100644
index 0000000..6ba8056
--- /dev/null
+++ b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/CipherUtility.java
@@ -0,0 +1,328 @@
+/*
+ * 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.security.util.crypto;
+
+import java.io.IOException;
+import java.io.InputStream;
+import java.io.OutputStream;
+import java.util.ArrayList;
+import java.util.Arrays;
+import java.util.Collections;
+import java.util.HashMap;
+import java.util.List;
+import java.util.Map;
+import java.util.regex.Matcher;
+import java.util.regex.Pattern;
+import javax.crypto.Cipher;
+import org.apache.commons.codec.binary.Base64;
+import org.apache.commons.lang3.StringUtils;
+import org.apache.nifi.processor.exception.ProcessException;
+import org.apache.nifi.security.util.EncryptionMethod;
+import org.apache.nifi.stream.io.ByteArrayOutputStream;
+import org.apache.nifi.stream.io.StreamUtils;
+
+public class CipherUtility {
+
+    public static final int BUFFER_SIZE = 65536;
+    private static final Pattern KEY_LENGTH_PATTERN = Pattern.compile("([\\d]+)BIT");
+
+    private static final Map<String, Integer> MAX_PASSWORD_LENGTH_BY_ALGORITHM;
+
+    static {
+        Map<String, Integer> aMap = new HashMap<>();
+        /**
+         * These values were determined empirically by running {@link NiFiLegacyCipherProviderGroovyTest#testShouldDetermineDependenceOnUnlimitedStrengthCrypto()}
+         *, which evaluates each algorithm in a try/catch harness with increasing password size until it throws an exception.
+         * This was performed on a JVM without the Unlimited Strength Jurisdiction cryptographic policy files installed.
+         */
+        aMap.put("PBEWITHMD5AND128BITAES-CBC-OPENSSL", 16);
+        aMap.put("PBEWITHMD5AND192BITAES-CBC-OPENSSL", 16);
+        aMap.put("PBEWITHMD5AND256BITAES-CBC-OPENSSL", 16);
+        aMap.put("PBEWITHMD5ANDDES", 16);
+        aMap.put("PBEWITHMD5ANDRC2", 16);
+        aMap.put("PBEWITHSHA1ANDRC2", 16);
+        aMap.put("PBEWITHSHA1ANDDES", 16);
+        aMap.put("PBEWITHSHAAND128BITAES-CBC-BC", 7);
+        aMap.put("PBEWITHSHAAND192BITAES-CBC-BC", 7);
+        aMap.put("PBEWITHSHAAND256BITAES-CBC-BC", 7);
+        aMap.put("PBEWITHSHAAND40BITRC2-CBC", 7);
+        aMap.put("PBEWITHSHAAND128BITRC2-CBC", 7);
+        aMap.put("PBEWITHSHAAND40BITRC4", 7);
+        aMap.put("PBEWITHSHAAND128BITRC4", 7);
+        aMap.put("PBEWITHSHA256AND128BITAES-CBC-BC", 7);
+        aMap.put("PBEWITHSHA256AND192BITAES-CBC-BC", 7);
+        aMap.put("PBEWITHSHA256AND256BITAES-CBC-BC", 7);
+        aMap.put("PBEWITHSHAAND2-KEYTRIPLEDES-CBC", 7);
+        aMap.put("PBEWITHSHAAND3-KEYTRIPLEDES-CBC", 7);
+        aMap.put("PBEWITHSHAANDTWOFISH-CBC", 7);
+        MAX_PASSWORD_LENGTH_BY_ALGORITHM = Collections.unmodifiableMap(aMap);
+    }
+
+    /**
+     * Returns the cipher algorithm from the full algorithm name. Useful for getting key lengths, etc.
+     * <p/>
+     * Ex: PBEWITHMD5AND128BITAES-CBC-OPENSSL -> AES
+     *
+     * @param algorithm the full algorithm name
+     * @return the generic cipher name or the full algorithm if one cannot be extracted
+     */
+    public static String parseCipherFromAlgorithm(final String algorithm) {
+        if (StringUtils.isEmpty(algorithm)) {
+            return algorithm;
+        }
+        String formattedAlgorithm = algorithm.toUpperCase();
+
+        // This is not optimal but the algorithms do not have a standard format
+        final String AES = "AES";
+        final String TDES = "TRIPLEDES";
+        final String TDES_ALTERNATE = "DESEDE";
+        final String DES = "DES";
+        final String RC4 = "RC4";
+        final String RC2 = "RC2";
+        final String TWOFISH = "TWOFISH";
+        final List<String> SYMMETRIC_CIPHERS = Arrays.asList(AES, TDES, TDES_ALTERNATE, DES, RC4, RC2, TWOFISH);
+
+        // The algorithms contain "TRIPLEDES" but the cipher name is "DESede"
+        final String ACTUAL_TDES_CIPHER = "DESede";
+
+        for (String cipher : SYMMETRIC_CIPHERS) {
+            if (formattedAlgorithm.contains(cipher)) {
+                if (cipher.equals(TDES) || cipher.equals(TDES_ALTERNATE)) {
+                    return ACTUAL_TDES_CIPHER;
+                } else {
+                    return cipher;
+                }
+            }
+        }
+
+        return algorithm;
+    }
+
+    /**
+     * Returns the cipher key length from the full algorithm name. Useful for getting key lengths, etc.
+     * <p/>
+     * Ex: PBEWITHMD5AND128BITAES-CBC-OPENSSL -> 128
+     *
+     * @param algorithm the full algorithm name
+     * @return the key length or -1 if one cannot be extracted
+     */
+    public static int parseKeyLengthFromAlgorithm(final String algorithm) {
+        int keyLength = parseActualKeyLengthFromAlgorithm(algorithm);
+        if (keyLength != -1) {
+            return keyLength;
+        } else {
+            // Key length not explicitly named in algorithm
+            String cipher = parseCipherFromAlgorithm(algorithm);
+            return getDefaultKeyLengthForCipher(cipher);
+        }
+    }
+
+    private static int parseActualKeyLengthFromAlgorithm(final String algorithm) {
+        Matcher matcher = KEY_LENGTH_PATTERN.matcher(algorithm);
+        if (matcher.find()) {
+            return Integer.parseInt(matcher.group(1));
+        } else {
+            return -1;
+        }
+    }
+
+    /**
+     * Returns true if the provided key length is a valid key length for the provided cipher family. Does not reflect if the Unlimited Strength Cryptography Jurisdiction Policies are installed.
+     * Does not reflect if the key length is correct for a specific combination of cipher and PBE-derived key length.
+     * <p/>
+     * Ex:
+     * <p/>
+     * 256 is valid for {@code AES/CBC/PKCS7Padding} but not {@code PBEWITHMD5AND128BITAES-CBC-OPENSSL}. However, this method will return {@code true} for both because it only gets the cipher
+     * family, {@code AES}.
+     * <p/>
+     * 64, AES -> false
+     * [128, 192, 256], AES -> true
+     *
+     * @param keyLength the key length in bits
+     * @param cipher    the cipher family
+     * @return true if this key length is valid
+     */
+    public static boolean isValidKeyLength(int keyLength, final String cipher) {
+        if (StringUtils.isEmpty(cipher)) {
+            return false;
+        }
+        return getValidKeyLengthsForAlgorithm(cipher).contains(keyLength);
+    }
+
+    /**
+     * Returns true if the provided key length is a valid key length for the provided algorithm. Does not reflect if the Unlimited Strength Cryptography Jurisdiction Policies are installed.
+     * <p/>
+     * Ex:
+     * <p/>
+     * 256 is valid for {@code AES/CBC/PKCS7Padding} but not {@code PBEWITHMD5AND128BITAES-CBC-OPENSSL}.
+     * <p/>
+     * 64, AES/CBC/PKCS7Padding -> false
+     * [128, 192, 256], AES/CBC/PKCS7Padding -> true
+     * <p/>
+     * 128, PBEWITHMD5AND128BITAES-CBC-OPENSSL -> true
+     * [192, 256], PBEWITHMD5AND128BITAES-CBC-OPENSSL -> false
+     *
+     * @param keyLength the key length in bits
+     * @param algorithm the specific algorithm
+     * @return true if this key length is valid
+     */
+    public static boolean isValidKeyLengthForAlgorithm(int keyLength, final String algorithm) {
+        if (StringUtils.isEmpty(algorithm)) {
+            return false;
+        }
+        return getValidKeyLengthsForAlgorithm(algorithm).contains(keyLength);
+    }
+
+    public static List<Integer> getValidKeyLengthsForAlgorithm(String algorithm) {
+        List<Integer> validKeyLengths = new ArrayList<>();
+        if (StringUtils.isEmpty(algorithm)) {
+            return validKeyLengths;
+        }
+
+        // Some algorithms specify a single key size
+        int keyLength = parseActualKeyLengthFromAlgorithm(algorithm);
+        if (keyLength != -1) {
+            validKeyLengths.add(keyLength);
+            return validKeyLengths;
+        }
+
+        // The algorithm does not specify a key size
+        String cipher = parseCipherFromAlgorithm(algorithm);
+        switch (cipher.toUpperCase()) {
+            case "DESEDE":
+                // 3DES keys have the cryptographic strength of 7/8 because of parity bits, but are often represented with n*8 bytes
+                return Arrays.asList(56, 64, 112, 128, 168, 192);
+            case "DES":
+                return Arrays.asList(56, 64);
+            case "RC2":
+            case "RC4":
+            case "RC5":
+                /** These ciphers can have arbitrary length keys but that's a really bad idea, {@see http://crypto.stackexchange.com/a/9963/12569}.
+                 * Also, RC* is deprecated and should be considered insecure */
+                for (int i = 40; i <= 2048; i++) {
+                    validKeyLengths.add(i);
+                }
+                return validKeyLengths;
+            case "AES":
+            case "TWOFISH":
+                return Arrays.asList(128, 192, 256);
+            default:
+                return validKeyLengths;
+        }
+    }
+
+    private static int getDefaultKeyLengthForCipher(String cipher) {
+        if (StringUtils.isEmpty(cipher)) {
+            return -1;
+        }
+        cipher = cipher.toUpperCase();
+        switch (cipher) {
+            case "DESEDE":
+                return 112;
+            case "DES":
+                return 64;
+            case "RC2":
+            case "RC4":
+            case "RC5":
+            default:
+                return 128;
+        }
+    }
+
+    public static void processStreams(Cipher cipher, InputStream in, OutputStream out) {
+        try {
+            final byte[] buffer = new byte[BUFFER_SIZE];
+            int len;
+            while ((len = in.read(buffer)) > 0) {
+                final byte[] decryptedBytes = cipher.update(buffer, 0, len);
+                if (decryptedBytes != null) {
+                    out.write(decryptedBytes);
+                }
+            }
+
+            out.write(cipher.doFinal());
+        } catch (Exception e) {
+            throw new ProcessException(e);
+        }
+    }
+
+    public static byte[] readBytesFromInputStream(InputStream in, String label, int limit, byte[] delimiter) throws IOException, ProcessException {
+        if (in == null) {
+            throw new IllegalArgumentException("Cannot read " + label + " from null InputStream");
+        }
+
+        // If the value is not detected within the first n bytes, throw an exception
+        in.mark(limit);
+
+        // The first n bytes of the input stream contain the value up to the custom delimiter
+        ByteArrayOutputStream bytesOut = new ByteArrayOutputStream();
+        byte[] stoppedBy = StreamUtils.copyExclusive(in, bytesOut, limit + delimiter.length, delimiter);
+
+        if (stoppedBy != null) {
+            byte[] bytes = bytesOut.toByteArray();
+            return bytes;
+        }
+
+        // If no delimiter was found, reset the cursor
+        in.reset();
+        return null;
+    }
+
+    public static void writeBytesToOutputStream(OutputStream out, byte[] value, String label, byte[] delimiter) throws IOException {
+        if (out == null) {
+            throw new IllegalArgumentException("Cannot write " + label + " to null OutputStream");
+        }
+        out.write(value);
+        out.write(delimiter);
+    }
+
+    public static String encodeBase64NoPadding(final byte[] bytes) {
+        String base64UrlNoPadding = Base64.encodeBase64URLSafeString(bytes);
+        base64UrlNoPadding = base64UrlNoPadding.replaceAll("-", "+");
+        base64UrlNoPadding = base64UrlNoPadding.replaceAll("_", "/");
+        return base64UrlNoPadding;
+    }
+
+    public static boolean passwordLengthIsValidForAlgorithmOnLimitedStrengthCrypto(final int passwordLength, EncryptionMethod encryptionMethod) {
+        if (encryptionMethod == null) {
+            throw new IllegalArgumentException("Cannot evaluate an empty encryption method algorithm");
+        }
+
+        return passwordLength <= getMaximumPasswordLengthForAlgorithmOnLimitedStrengthCrypto(encryptionMethod);
+    }
+
+    public static int getMaximumPasswordLengthForAlgorithmOnLimitedStrengthCrypto(EncryptionMethod encryptionMethod) {
+        if (encryptionMethod == null) {
+            throw new IllegalArgumentException("Cannot evaluate an empty encryption method algorithm");
+        }
+
+        if (MAX_PASSWORD_LENGTH_BY_ALGORITHM.containsKey(encryptionMethod.getAlgorithm())) {
+            return MAX_PASSWORD_LENGTH_BY_ALGORITHM.get(encryptionMethod.getAlgorithm());
+        } else {
+            return -1;
+        }
+    }
+
+    public static byte[] concatBytes(byte[]... arrays) throws IOException {
+        ByteArrayOutputStream outputStream = new ByteArrayOutputStream();
+        for (byte[] bytes : arrays) {
+            outputStream.write(bytes);
+        }
+
+        return outputStream.toByteArray();
+    }
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/KeyedCipherProvider.java
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/KeyedCipherProvider.java b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/KeyedCipherProvider.java
new file mode 100644
index 0000000..719150f
--- /dev/null
+++ b/nifi-commons/nifi-security-utils/src/main/java/org/apache/nifi/security/util/crypto/KeyedCipherProvider.java
@@ -0,0 +1,72 @@
+/*
+ * 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.security.util.crypto;
+
+import java.io.IOException;
+import java.io.InputStream;
+import java.io.OutputStream;
+import java.nio.charset.StandardCharsets;
+import javax.crypto.Cipher;
+import javax.crypto.SecretKey;
+import org.apache.nifi.processor.exception.ProcessException;
+import org.apache.nifi.security.util.EncryptionMethod;
+
+public abstract class KeyedCipherProvider implements CipherProvider {
+    static final byte[] IV_DELIMITER = "NiFiIV".getBytes(StandardCharsets.UTF_8);
+    // This is 16 bytes for AES but can vary for other ciphers
+    static final int MAX_IV_LIMIT = 16;
+
+    /**
+     * Returns an initialized cipher for the specified algorithm. The IV is provided externally to allow for non-deterministic IVs, as IVs
+     * deterministically derived from the password are a potential vulnerability and compromise semantic security. See
+     * <a href="http://crypto.stackexchange.com/a/3970/12569">Ilmari Karonen's answer on Crypto Stack Exchange</a>
+     *
+     * @param encryptionMethod the {@link EncryptionMethod}
+     * @param key              the key
+     * @param iv               the IV or nonce
+     * @param encryptMode      true for encrypt, false for decrypt
+     * @return the initialized cipher
+     * @throws Exception if there is a problem initializing the cipher
+     */
+    abstract Cipher getCipher(EncryptionMethod encryptionMethod, SecretKey key, byte[] iv, boolean encryptMode) throws Exception;
+
+    /**
+     * Returns an initialized cipher for the specified algorithm. The IV will be generated internally (for encryption). If decryption is requested, it will throw an exception.
+     *
+     * @param encryptionMethod the {@link EncryptionMethod}
+     * @param key              the key
+     * @param encryptMode      true for encrypt, false for decrypt
+     * @return the initialized cipher
+     * @throws Exception if there is a problem initializing the cipher or if decryption is requested
+     */
+    abstract Cipher getCipher(EncryptionMethod encryptionMethod, SecretKey key, boolean encryptMode) throws Exception;
+
+    /**
+     * Generates a new random IV of the correct length.
+     *
+     * @return the IV
+     */
+    abstract byte[] generateIV();
+
+    public byte[] readIV(InputStream in) throws IOException, ProcessException {
+        return CipherUtility.readBytesFromInputStream(in, "IV", MAX_IV_LIMIT, IV_DELIMITER);
+    }
+
+    public void writeIV(byte[] iv, OutputStream out) throws IOException {
+        CipherUtility.writeBytesToOutputStream(out, iv, "IV", IV_DELIMITER);
+    }
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/AESKeyedCipherProviderGroovyTest.groovy
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/AESKeyedCipherProviderGroovyTest.groovy b/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/AESKeyedCipherProviderGroovyTest.groovy
new file mode 100644
index 0000000..8082149
--- /dev/null
+++ b/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/AESKeyedCipherProviderGroovyTest.groovy
@@ -0,0 +1,347 @@
+/*
+ * 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.security.util.crypto
+
+import org.apache.commons.codec.binary.Hex
+import org.apache.nifi.security.util.EncryptionMethod
+import org.bouncycastle.jce.provider.BouncyCastleProvider
+import org.junit.After
+import org.junit.Assume
+import org.junit.Before
+import org.junit.BeforeClass
+import org.junit.Test
+import org.junit.runner.RunWith
+import org.junit.runners.JUnit4
+import org.slf4j.Logger
+import org.slf4j.LoggerFactory
+
+import javax.crypto.Cipher
+import javax.crypto.SecretKey
+import javax.crypto.spec.SecretKeySpec
+import java.security.SecureRandom
+import java.security.Security
+
+import static groovy.test.GroovyAssert.shouldFail
+
+@RunWith(JUnit4.class)
+class AESKeyedCipherProviderGroovyTest {
+    private static final Logger logger = LoggerFactory.getLogger(AESKeyedCipherProviderGroovyTest.class)
+
+    private static final String KEY_HEX = "0123456789ABCDEFFEDCBA9876543210"
+
+    private static final List<EncryptionMethod> keyedEncryptionMethods = EncryptionMethod.values().findAll { it.keyedCipher }
+
+    private static final SecretKey key = new SecretKeySpec(Hex.decodeHex(KEY_HEX as char[]), "AES")
+
+    @BeforeClass
+    static void setUpOnce() throws Exception {
+        Security.addProvider(new BouncyCastleProvider())
+
+        logger.metaClass.methodMissing = { String name, args ->
+            logger.info("[${name?.toUpperCase()}] ${(args as List).join(" ")}")
+        }
+    }
+
+    @Before
+    void setUp() throws Exception {
+    }
+
+    @After
+    void tearDown() throws Exception {
+    }
+
+    private static boolean isUnlimitedStrengthCryptoAvailable() {
+        Cipher.getMaxAllowedKeyLength("AES") > 128
+    }
+
+    @Test
+    void testGetCipherShouldBeInternallyConsistent() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        final String plaintext = "This is a plaintext message."
+
+        // Act
+        for (EncryptionMethod em : keyedEncryptionMethods) {
+            logger.info("Using algorithm: ${em.getAlgorithm()}")
+
+            // Initialize a cipher for encryption
+            Cipher cipher = cipherProvider.getCipher(em, key, true)
+            byte[] iv = cipher.getIV()
+            logger.info("IV: ${Hex.encodeHexString(iv)}")
+
+            byte[] cipherBytes = cipher.doFinal(plaintext.getBytes("UTF-8"))
+            logger.info("Cipher text: ${Hex.encodeHexString(cipherBytes)} ${cipherBytes.length}")
+
+            cipher = cipherProvider.getCipher(em, key, iv, false)
+            byte[] recoveredBytes = cipher.doFinal(cipherBytes)
+            String recovered = new String(recoveredBytes, "UTF-8")
+            logger.info("Recovered: ${recovered}")
+
+            // Assert
+            assert plaintext.equals(recovered)
+        }
+    }
+
+    @Test
+    void testGetCipherWithExternalIVShouldBeInternallyConsistent() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        final String plaintext = "This is a plaintext message."
+
+        // Act
+        keyedEncryptionMethods.each { EncryptionMethod em ->
+            logger.info("Using algorithm: ${em.getAlgorithm()}")
+            byte[] iv = cipherProvider.generateIV()
+            logger.info("IV: ${Hex.encodeHexString(iv)}")
+
+            // Initialize a cipher for encryption
+            Cipher cipher = cipherProvider.getCipher(em, key, iv, true)
+
+            byte[] cipherBytes = cipher.doFinal(plaintext.getBytes("UTF-8"))
+            logger.info("Cipher text: ${Hex.encodeHexString(cipherBytes)} ${cipherBytes.length}")
+
+            cipher = cipherProvider.getCipher(em, key, iv, false)
+            byte[] recoveredBytes = cipher.doFinal(cipherBytes)
+            String recovered = new String(recoveredBytes, "UTF-8")
+            logger.info("Recovered: ${recovered}")
+
+            // Assert
+            assert plaintext.equals(recovered)
+        }
+    }
+
+    @Test
+    void testGetCipherWithUnlimitedStrengthShouldBeInternallyConsistent() throws Exception {
+        // Arrange
+        Assume.assumeTrue("Test is being skipped due to this JVM lacking JCE Unlimited Strength Jurisdiction Policy file.", isUnlimitedStrengthCryptoAvailable())
+
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+        final List<Integer> LONG_KEY_LENGTHS = [192, 256]
+
+        final String plaintext = "This is a plaintext message."
+
+        SecureRandom secureRandom = new SecureRandom()
+
+        // Act
+        keyedEncryptionMethods.each { EncryptionMethod em ->
+            // Re-use the same IV for the different length keys to ensure the encryption is different
+            byte[] iv = cipherProvider.generateIV()
+            logger.info("IV: ${Hex.encodeHexString(iv)}")
+
+            LONG_KEY_LENGTHS.each { int keyLength ->
+                logger.info("Using algorithm: ${em.getAlgorithm()} with key length ${keyLength}")
+
+                // Generate a key
+                byte[] keyBytes = new byte[keyLength / 8]
+                secureRandom.nextBytes(keyBytes)
+                SecretKey localKey = new SecretKeySpec(keyBytes, "AES")
+                logger.info("Key: ${Hex.encodeHexString(keyBytes)} ${keyBytes.length}")
+
+                // Initialize a cipher for encryption
+                Cipher cipher = cipherProvider.getCipher(em, localKey, iv, true)
+
+                byte[] cipherBytes = cipher.doFinal(plaintext.getBytes("UTF-8"))
+                logger.info("Cipher text: ${Hex.encodeHexString(cipherBytes)} ${cipherBytes.length}")
+
+                cipher = cipherProvider.getCipher(em, localKey, iv, false)
+                byte[] recoveredBytes = cipher.doFinal(cipherBytes)
+                String recovered = new String(recoveredBytes, "UTF-8")
+                logger.info("Recovered: ${recovered}")
+
+                // Assert
+                assert plaintext.equals(recovered)
+            }
+        }
+    }
+
+    @Test
+    void testShouldRejectEmptyKey() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        final EncryptionMethod encryptionMethod = EncryptionMethod.AES_CBC
+
+        // Act
+        def msg = shouldFail(IllegalArgumentException) {
+            cipherProvider.getCipher(encryptionMethod, null, true)
+        }
+
+        // Assert
+        assert msg =~ "The key must be specified"
+    }
+
+    @Test
+    void testShouldRejectIncorrectLengthKey() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        SecretKey localKey = new SecretKeySpec(Hex.decodeHex("0123456789ABCDEF" as char[]), "AES")
+        assert ![128, 192, 256].contains(localKey.encoded.length)
+
+        final EncryptionMethod encryptionMethod = EncryptionMethod.AES_CBC
+
+        // Act
+        def msg = shouldFail(IllegalArgumentException) {
+            cipherProvider.getCipher(encryptionMethod, localKey, true)
+        }
+
+        // Assert
+        assert msg =~ "The key must be of length \\[128, 192, 256\\]"
+    }
+
+    @Test
+    void testShouldRejectEmptyEncryptionMethod() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        // Act
+        def msg = shouldFail(IllegalArgumentException) {
+            cipherProvider.getCipher(null, key, true)
+        }
+
+        // Assert
+        assert msg =~ "The encryption method must be specified"
+    }
+
+    @Test
+    void testShouldRejectUnsupportedEncryptionMethod() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        final EncryptionMethod encryptionMethod = EncryptionMethod.MD5_128AES
+
+        // Act
+        def msg = shouldFail(IllegalArgumentException) {
+            cipherProvider.getCipher(encryptionMethod, key, true)
+        }
+
+        // Assert
+        assert msg =~ " requires a PBECipherProvider"
+    }
+
+    @Test
+    void testGetCipherShouldSupportExternalCompatibility() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        final String PLAINTEXT = "This is a plaintext message."
+
+        // These values can be generated by running `$ ./openssl_aes.rb` in the terminal
+        final byte[] IV = Hex.decodeHex("e0bc8cc7fbc0bdfdc184dc22ce2fcb5b" as char[])
+        final byte[] LOCAL_KEY = Hex.decodeHex("c72943d27c3e5a276169c5998a779117" as char[])
+        final String CIPHER_TEXT = "a2725ea55c7dd717664d044cab0f0b5f763653e322c27df21954f5be394efb1b"
+        byte[] cipherBytes = Hex.decodeHex(CIPHER_TEXT as char[])
+
+        SecretKey localKey = new SecretKeySpec(LOCAL_KEY, "AES")
+
+        EncryptionMethod encryptionMethod = EncryptionMethod.AES_CBC
+        logger.info("Using algorithm: ${encryptionMethod.getAlgorithm()}")
+        logger.info("Cipher text: ${Hex.encodeHexString(cipherBytes)} ${cipherBytes.length}")
+
+        // Act
+        Cipher cipher = cipherProvider.getCipher(encryptionMethod, localKey, IV, false)
+        byte[] recoveredBytes = cipher.doFinal(cipherBytes)
+        String recovered = new String(recoveredBytes, "UTF-8")
+        logger.info("Recovered: ${recovered}")
+
+        // Assert
+        assert PLAINTEXT.equals(recovered)
+    }
+
+    @Test
+    void testGetCipherForDecryptShouldRequireIV() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        final String plaintext = "This is a plaintext message."
+
+        // Act
+        keyedEncryptionMethods.each { EncryptionMethod em ->
+            logger.info("Using algorithm: ${em.getAlgorithm()}")
+            byte[] iv = cipherProvider.generateIV()
+            logger.info("IV: ${Hex.encodeHexString(iv)}")
+
+            // Initialize a cipher for encryption
+            Cipher cipher = cipherProvider.getCipher(em, key, iv, true)
+
+            byte[] cipherBytes = cipher.doFinal(plaintext.getBytes("UTF-8"))
+            logger.info("Cipher text: ${Hex.encodeHexString(cipherBytes)} ${cipherBytes.length}")
+
+            def msg = shouldFail(IllegalArgumentException) {
+                cipher = cipherProvider.getCipher(em, key, false)
+            }
+
+            // Assert
+            assert msg =~ "Cannot decrypt without a valid IV"
+        }
+    }
+
+    @Test
+    void testGetCipherShouldRejectInvalidIVLengths() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        final def INVALID_IVS = (0..15).collect { int length -> new byte[length] }
+
+        EncryptionMethod encryptionMethod = EncryptionMethod.AES_CBC
+
+        // Act
+        INVALID_IVS.each { byte[] badIV ->
+            logger.info("IV: ${Hex.encodeHexString(badIV)} ${badIV.length}")
+
+            // Encrypt should print a warning about the bad IV but overwrite it
+            Cipher cipher = cipherProvider.getCipher(encryptionMethod, key, badIV, true)
+
+            // Decrypt should fail
+            def msg = shouldFail(IllegalArgumentException) {
+                cipher = cipherProvider.getCipher(encryptionMethod, key, badIV, false)
+            }
+            logger.expected(msg)
+
+            // Assert
+            assert msg =~ "Cannot decrypt without a valid IV"
+        }
+    }
+
+    @Test
+    void testGetCipherShouldRejectEmptyIV() throws Exception {
+        // Arrange
+        KeyedCipherProvider cipherProvider = new AESKeyedCipherProvider()
+
+        EncryptionMethod encryptionMethod = EncryptionMethod.AES_CBC
+
+        byte[] badIV = [0x00 as byte] * 16 as byte[]
+
+        // Act
+        logger.info("IV: ${Hex.encodeHexString(badIV)} ${badIV.length}")
+
+        // Encrypt should print a warning about the bad IV but overwrite it
+        Cipher cipher = cipherProvider.getCipher(encryptionMethod, key, badIV, true)
+        logger.info("IV after encrypt: ${Hex.encodeHexString(cipher.getIV())}")
+
+        // Decrypt should fail
+        def msg = shouldFail(IllegalArgumentException) {
+            cipher = cipherProvider.getCipher(encryptionMethod, key, badIV, false)
+        }
+        logger.expected(msg)
+
+        // Assert
+        assert msg =~ "Cannot decrypt without a valid IV"
+    }
+}
\ No newline at end of file

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/CipherUtilityGroovyTest.groovy
----------------------------------------------------------------------
diff --git a/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/CipherUtilityGroovyTest.groovy b/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/CipherUtilityGroovyTest.groovy
new file mode 100644
index 0000000..8f092f3
--- /dev/null
+++ b/nifi-commons/nifi-security-utils/src/test/groovy/org/apache/nifi/security/util/crypto/CipherUtilityGroovyTest.groovy
@@ -0,0 +1,251 @@
+/*
+ * 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.security.util.crypto
+
+import org.apache.nifi.security.util.EncryptionMethod
+import org.bouncycastle.jce.provider.BouncyCastleProvider
+import org.junit.After
+import org.junit.Before
+import org.junit.BeforeClass
+import org.junit.Test
+import org.junit.runner.RunWith
+import org.junit.runners.JUnit4
+import org.slf4j.Logger
+import org.slf4j.LoggerFactory
+
+import java.security.Security
+
+@RunWith(JUnit4.class)
+class CipherUtilityGroovyTest extends GroovyTestCase {
+    private static final Logger logger = LoggerFactory.getLogger(CipherUtilityGroovyTest.class)
+
+    // TripleDES must precede DES for automatic grouping precedence
+    private static final List<String> CIPHERS = ["AES", "TRIPLEDES", "DES", "RC2", "RC4", "RC5", "TWOFISH"]
+    private static final List<String> SYMMETRIC_ALGORITHMS = EncryptionMethod.values().findAll { it.algorithm.startsWith("PBE") || it.algorithm.startsWith("AES") }*.algorithm
+    private static final Map<String, List<String>> ALGORITHMS_MAPPED_BY_CIPHER = SYMMETRIC_ALGORITHMS.groupBy { String algorithm -> CIPHERS.find { algorithm.contains(it) } }
+
+    // Manually mapped as of 01/19/16 0.5.0
+    private static final Map<Integer, List<String>> ALGORITHMS_MAPPED_BY_KEY_LENGTH = [
+            (40) : ["PBEWITHSHAAND40BITRC2-CBC",
+                    "PBEWITHSHAAND40BITRC4"],
+            (64) : ["PBEWITHMD5ANDDES",
+                    "PBEWITHSHA1ANDDES"],
+            (112): ["PBEWITHSHAAND2-KEYTRIPLEDES-CBC",
+                    "PBEWITHSHAAND3-KEYTRIPLEDES-CBC"],
+            (128): ["PBEWITHMD5AND128BITAES-CBC-OPENSSL",
+                    "PBEWITHMD5ANDRC2",
+                    "PBEWITHSHA1ANDRC2",
+                    "PBEWITHSHA256AND128BITAES-CBC-BC",
+                    "PBEWITHSHAAND128BITAES-CBC-BC",
+                    "PBEWITHSHAAND128BITRC2-CBC",
+                    "PBEWITHSHAAND128BITRC4",
+                    "PBEWITHSHAANDTWOFISH-CBC",
+                    "AES/CBC/PKCS7Padding",
+                    "AES/CTR/NoPadding",
+                    "AES/GCM/NoPadding"],
+            (192): ["PBEWITHMD5AND192BITAES-CBC-OPENSSL",
+                    "PBEWITHSHA256AND192BITAES-CBC-BC",
+                    "PBEWITHSHAAND192BITAES-CBC-BC",
+                    "AES/CBC/PKCS7Padding",
+                    "AES/CTR/NoPadding",
+                    "AES/GCM/NoPadding"],
+            (256): ["PBEWITHMD5AND256BITAES-CBC-OPENSSL",
+                    "PBEWITHSHA256AND256BITAES-CBC-BC",
+                    "PBEWITHSHAAND256BITAES-CBC-BC",
+                    "AES/CBC/PKCS7Padding",
+                    "AES/CTR/NoPadding",
+                    "AES/GCM/NoPadding"]
+    ]
+
+    @BeforeClass
+    static void setUpOnce() {
+        Security.addProvider(new BouncyCastleProvider());
+
+        // Fix because TRIPLEDES -> DESede
+        def tripleDESAlgorithms = ALGORITHMS_MAPPED_BY_CIPHER.remove("TRIPLEDES")
+        ALGORITHMS_MAPPED_BY_CIPHER.put("DESede", tripleDESAlgorithms)
+
+        logger.info("Mapped algorithms: ${ALGORITHMS_MAPPED_BY_CIPHER}")
+    }
+
+    @Before
+    void setUp() throws Exception {
+
+    }
+
+    @After
+    void tearDown() throws Exception {
+
+    }
+
+    @Test
+    void testShouldParseCipherFromAlgorithm() {
+        // Arrange
+        final def EXPECTED_ALGORITHMS = ALGORITHMS_MAPPED_BY_CIPHER
+
+        // Act
+        SYMMETRIC_ALGORITHMS.each { String algorithm ->
+            String cipher = CipherUtility.parseCipherFromAlgorithm(algorithm)
+            logger.info("Extracted ${cipher} from ${algorithm}")
+
+            // Assert
+            assert EXPECTED_ALGORITHMS.get(cipher).contains(algorithm)
+        }
+    }
+
+    @Test
+    void testShouldParseKeyLengthFromAlgorithm() {
+        // Arrange
+        final def EXPECTED_ALGORITHMS = ALGORITHMS_MAPPED_BY_KEY_LENGTH
+
+        // Act
+        SYMMETRIC_ALGORITHMS.each { String algorithm ->
+            int keyLength = CipherUtility.parseKeyLengthFromAlgorithm(algorithm)
+            logger.info("Extracted ${keyLength} from ${algorithm}")
+
+            // Assert
+            assert EXPECTED_ALGORITHMS.get(keyLength).contains(algorithm)
+        }
+    }
+
+    @Test
+    void testShouldDetermineValidKeyLength() {
+        // Arrange
+
+        // Act
+        ALGORITHMS_MAPPED_BY_KEY_LENGTH.each { int keyLength, List<String> algorithms ->
+            algorithms.each { String algorithm ->
+                logger.info("Checking ${keyLength} for ${algorithm}")
+
+                // Assert
+                assert CipherUtility.isValidKeyLength(keyLength, CipherUtility.parseCipherFromAlgorithm(algorithm))
+            }
+        }
+    }
+
+    @Test
+    void testShouldDetermineInvalidKeyLength() {
+        // Arrange
+
+        // Act
+        ALGORITHMS_MAPPED_BY_KEY_LENGTH.each { int keyLength, List<String> algorithms ->
+            algorithms.each { String algorithm ->
+                def invalidKeyLengths = [-1, 0, 1]
+                if (algorithm =~ "RC\\d") {
+                    invalidKeyLengths += [39, 2049]
+                } else {
+                    invalidKeyLengths += keyLength + 1
+                }
+                logger.info("Checking ${invalidKeyLengths.join(", ")} for ${algorithm}")
+
+                // Assert
+                invalidKeyLengths.each { int invalidKeyLength ->
+                    assert !CipherUtility.isValidKeyLength(invalidKeyLength, CipherUtility.parseCipherFromAlgorithm(algorithm))
+                }
+            }
+        }
+    }
+
+    @Test
+    void testShouldDetermineValidKeyLengthForAlgorithm() {
+        // Arrange
+
+        // Act
+        ALGORITHMS_MAPPED_BY_KEY_LENGTH.each { int keyLength, List<String> algorithms ->
+            algorithms.each { String algorithm ->
+                logger.info("Checking ${keyLength} for ${algorithm}")
+
+                // Assert
+                assert CipherUtility.isValidKeyLengthForAlgorithm(keyLength, algorithm)
+            }
+        }
+    }
+
+    @Test
+    void testShouldDetermineInvalidKeyLengthForAlgorithm() {
+        // Arrange
+
+        // Act
+        ALGORITHMS_MAPPED_BY_KEY_LENGTH.each { int keyLength, List<String> algorithms ->
+            algorithms.each { String algorithm ->
+                def invalidKeyLengths = [-1, 0, 1]
+                if (algorithm =~ "RC\\d") {
+                    invalidKeyLengths += [39, 2049]
+                } else {
+                    invalidKeyLengths += keyLength + 1
+                }
+                logger.info("Checking ${invalidKeyLengths.join(", ")} for ${algorithm}")
+
+                // Assert
+                invalidKeyLengths.each { int invalidKeyLength ->
+                    assert !CipherUtility.isValidKeyLengthForAlgorithm(invalidKeyLength, algorithm)
+                }
+            }
+        }
+
+        // Extra hard-coded checks
+        String algorithm = "PBEWITHSHA256AND256BITAES-CBC-BC"
+        int invalidKeyLength = 192
+        logger.info("Checking ${invalidKeyLength} for ${algorithm}")
+        assert !CipherUtility.isValidKeyLengthForAlgorithm(invalidKeyLength, algorithm)
+    }
+
+    @Test
+    void testShouldGetValidKeyLengthsForAlgorithm() {
+        // Arrange
+
+        def rcKeyLengths = (40..2048).asList()
+        def CIPHER_KEY_SIZES = [
+                AES    : [128, 192, 256],
+                DES    : [56, 64],
+                DESede : [56, 64, 112, 128, 168, 192],
+                RC2    : rcKeyLengths,
+                RC4    : rcKeyLengths,
+                RC5    : rcKeyLengths,
+                TWOFISH: [128, 192, 256]
+        ]
+
+        def SINGLE_KEY_SIZE_ALGORITHMS = EncryptionMethod.values()*.algorithm.findAll { CipherUtility.parseActualKeyLengthFromAlgorithm(it) != -1 }
+        logger.info("Single key size algorithms: ${SINGLE_KEY_SIZE_ALGORITHMS}")
+        def MULTIPLE_KEY_SIZE_ALGORITHMS = EncryptionMethod.values()*.algorithm - SINGLE_KEY_SIZE_ALGORITHMS
+        MULTIPLE_KEY_SIZE_ALGORITHMS.removeAll { it.contains("PGP") }
+        logger.info("Multiple key size algorithms: ${MULTIPLE_KEY_SIZE_ALGORITHMS}")
+
+        // Act
+        SINGLE_KEY_SIZE_ALGORITHMS.each { String algorithm ->
+            def EXPECTED_KEY_SIZES = [CipherUtility.parseKeyLengthFromAlgorithm(algorithm)]
+
+            def validKeySizes = CipherUtility.getValidKeyLengthsForAlgorithm(algorithm)
+            logger.info("Checking ${algorithm} ${validKeySizes} against expected ${EXPECTED_KEY_SIZES}")
+
+            // Assert
+            assert validKeySizes == EXPECTED_KEY_SIZES
+        }
+
+        // Act
+        MULTIPLE_KEY_SIZE_ALGORITHMS.each { String algorithm ->
+            String cipher = CipherUtility.parseCipherFromAlgorithm(algorithm)
+            def EXPECTED_KEY_SIZES = CIPHER_KEY_SIZES[cipher]
+
+            def validKeySizes = CipherUtility.getValidKeyLengthsForAlgorithm(algorithm)
+            logger.info("Checking ${algorithm} ${validKeySizes} against expected ${EXPECTED_KEY_SIZES}")
+
+            // Assert
+            assert validKeySizes == EXPECTED_KEY_SIZES
+        }
+    }
+}

http://git-wip-us.apache.org/repos/asf/nifi/blob/7d242076/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java
----------------------------------------------------------------------
diff --git a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java
index 7853591..5b2ea41 100644
--- a/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java
+++ b/nifi-nar-bundles/nifi-framework-bundle/nifi-framework/nifi-framework-core/src/main/java/org/apache/nifi/controller/FlowController.java
@@ -16,7 +16,38 @@
  */
 package org.apache.nifi.controller;
 
+import static java.util.Objects.requireNonNull;
+
 import com.sun.jersey.api.client.ClientHandlerException;
+import java.io.ByteArrayInputStream;
+import java.io.IOException;
+import java.io.InputStream;
+import java.io.OutputStream;
+import java.net.URL;
+import java.util.ArrayList;
+import java.util.Collection;
+import java.util.Collections;
+import java.util.Date;
+import java.util.HashMap;
+import java.util.HashSet;
+import java.util.LinkedHashSet;
+import java.util.List;
+import java.util.Map;
+import java.util.Set;
+import java.util.UUID;
+import java.util.concurrent.ConcurrentHashMap;
+import java.util.concurrent.ConcurrentMap;
+import java.util.concurrent.Future;
+import java.util.concurrent.ScheduledExecutorService;
+import java.util.concurrent.ScheduledFuture;
+import java.util.concurrent.TimeUnit;
+import java.util.concurrent.atomic.AtomicBoolean;
+import java.util.concurrent.atomic.AtomicInteger;
+import java.util.concurrent.atomic.AtomicReference;
+import java.util.concurrent.locks.Lock;
+import java.util.concurrent.locks.ReentrantReadWriteLock;
+import java.util.stream.Collectors;
+import javax.net.ssl.SSLContext;
 import org.apache.commons.collections4.Predicate;
 import org.apache.commons.lang3.StringUtils;
 import org.apache.nifi.action.Action;
@@ -85,7 +116,6 @@ import org.apache.nifi.controller.repository.FlowFileRecord;
 import org.apache.nifi.controller.repository.FlowFileRepository;
 import org.apache.nifi.controller.repository.FlowFileSwapManager;
 import org.apache.nifi.controller.repository.QueueProvider;
-import org.apache.nifi.controller.repository.RepositoryRecord;
 import org.apache.nifi.controller.repository.RepositoryStatusReport;
 import org.apache.nifi.controller.repository.StandardCounterRepository;
 import org.apache.nifi.controller.repository.StandardFlowFileRecord;
@@ -217,38 +247,6 @@ import org.apache.zookeeper.server.quorum.QuorumPeerConfig.ConfigException;
 import org.slf4j.Logger;
 import org.slf4j.LoggerFactory;
 
-import javax.net.ssl.SSLContext;
-import java.io.ByteArrayInputStream;
-import java.io.IOException;
-import java.io.InputStream;
-import java.io.OutputStream;
-import java.net.URL;
-import java.util.ArrayList;
-import java.util.Collection;
-import java.util.Collections;
-import java.util.Date;
-import java.util.HashMap;
-import java.util.HashSet;
-import java.util.LinkedHashSet;
-import java.util.List;
-import java.util.Map;
-import java.util.Set;
-import java.util.UUID;
-import java.util.concurrent.ConcurrentHashMap;
-import java.util.concurrent.ConcurrentMap;
-import java.util.concurrent.Future;
-import java.util.concurrent.ScheduledExecutorService;
-import java.util.concurrent.ScheduledFuture;
-import java.util.concurrent.TimeUnit;
-import java.util.concurrent.atomic.AtomicBoolean;
-import java.util.concurrent.atomic.AtomicInteger;
-import java.util.concurrent.atomic.AtomicReference;
-import java.util.concurrent.locks.Lock;
-import java.util.concurrent.locks.ReentrantReadWriteLock;
-import java.util.stream.Collectors;
-
-import static java.util.Objects.requireNonNull;
-
 public class FlowController implements EventAccess, ControllerServiceProvider, ReportingTaskProvider,
     QueueProvider, Authorizable, ProvenanceAuthorizableFactory, NodeTypeProvider, IdentifierLookup, ReloadComponent {
 
@@ -3841,7 +3839,7 @@ public class FlowController implements EventAccess, ControllerServiceProvider, R
         final ProvenanceEventRecord sendEvent = new StandardProvenanceEventRecord.Builder()
                 .setEventType(ProvenanceEventType.DOWNLOAD)
                 .setFlowFileUUID(provEvent.getFlowFileUuid())
-                .setAttributes(provEvent.getAttributes(), Collections.<String, String>emptyMap())
+                .setAttributes(provEvent.getAttributes(), Collections.emptyMap())
                 .setCurrentContentClaim(resourceClaim.getContainer(), resourceClaim.getSection(), resourceClaim.getId(), offset, size)
                 .setTransitUri(requestUri)
                 .setEventTime(System.currentTimeMillis())
@@ -3883,7 +3881,7 @@ public class FlowController implements EventAccess, ControllerServiceProvider, R
         final StandardProvenanceEventRecord.Builder sendEventBuilder = new StandardProvenanceEventRecord.Builder()
                 .setEventType(ProvenanceEventType.DOWNLOAD)
                 .setFlowFileUUID(flowFile.getAttribute(CoreAttributes.UUID.key()))
-                .setAttributes(flowFile.getAttributes(), Collections.<String, String>emptyMap())
+                .setAttributes(flowFile.getAttributes(), Collections.emptyMap())
                 .setTransitUri(requestUri)
                 .setEventTime(System.currentTimeMillis())
                 .setFlowFileEntryDate(flowFile.getEntryDate())
@@ -4062,7 +4060,7 @@ public class FlowController implements EventAccess, ControllerServiceProvider, R
                 .addChildUuid(newFlowFileUUID)
                 .addParentUuid(parentUUID)
                 .setFlowFileUUID(parentUUID)
-                .setAttributes(Collections.<String, String>emptyMap(), flowFileRecord.getAttributes())
+                .setAttributes(Collections.emptyMap(), flowFileRecord.getAttributes())
                 .setCurrentContentClaim(event.getContentClaimContainer(), event.getContentClaimSection(), event.getContentClaimIdentifier(), event.getContentClaimOffset(), event.getFileSize())
                 .setDetails("Replay requested by " + user.getIdentity())
                 .setEventTime(System.currentTimeMillis())
@@ -4077,7 +4075,7 @@ public class FlowController implements EventAccess, ControllerServiceProvider, R
         final StandardRepositoryRecord record = new StandardRepositoryRecord(queue);
         record.setWorking(flowFileRecord);
         record.setDestination(queue);
-        flowFileRepository.updateRepository(Collections.<RepositoryRecord>singleton(record));
+        flowFileRepository.updateRepository(Collections.singleton(record));
 
         // Enqueue the data
         queue.put(flowFileRecord);


Mime
View raw message