diff --git a/nifi-extension-bundles/nifi-aws-bundle/nifi-aws-processors/src/main/java/org/apache/nifi/processors/aws/kinesis/firehose/PutKinesisFirehose.java b/nifi-extension-bundles/nifi-aws-bundle/nifi-aws-processors/src/main/java/org/apache/nifi/processors/aws/kinesis/firehose/PutKinesisFirehose.java index 25d6e027c49c..fa2fa1396f96 100644 --- a/nifi-extension-bundles/nifi-aws-bundle/nifi-aws-processors/src/main/java/org/apache/nifi/processors/aws/kinesis/firehose/PutKinesisFirehose.java +++ b/nifi-extension-bundles/nifi-aws-bundle/nifi-aws-processors/src/main/java/org/apache/nifi/processors/aws/kinesis/firehose/PutKinesisFirehose.java @@ -134,8 +134,11 @@ public void onTrigger(final ProcessContext context, final ProcessSession session for (final FlowFile flowFile : flowFiles) { final String firehoseStreamName = context.getProperty(KINESIS_FIREHOSE_DELIVERY_STREAM_NAME).evaluateAttributeExpressions(flowFile).getValue(); - recordHash.computeIfAbsent(firehoseStreamName, k -> new ArrayList<>()); - session.read(flowFile, in -> recordHash.get(firehoseStreamName).add(Record.builder().data(SdkBytes.fromInputStream(in)).build())); + session.read(flowFile, in -> + recordHash + .computeIfAbsent(firehoseStreamName, k -> new ArrayList<>()) + .add(Record.builder().data(SdkBytes.fromInputStream(in)).build()) + ); final List flowFilesForStream = hashFlowFiles.computeIfAbsent(firehoseStreamName, k -> new ArrayList<>()); flowFilesForStream.add(flowFile);