markap14 commented on a change in pull request #4088: NIFI-7197 - In-place replacement in LookupRecord processor URL: https://github.com/apache/nifi/pull/4088#discussion_r387737802
########## File path: nifi-nar-bundles/nifi-standard-bundle/nifi-standard-processors/src/main/java/org/apache/nifi/processors/standard/LookupRecord.java ########## @@ -263,6 +288,70 @@ public void onPropertyModified(final PropertyDescriptor descriptor, final String protected Set<Relationship> route(final Record record, final RecordSchema writeSchema, final FlowFile flowFile, final ProcessContext context, final Tuple<Map<String, RecordPath>, RecordPath> flowFileContext) { + final boolean isInPlaceReplacement = context.getProperty(IN_PLACE_REPLACEMENT).asBoolean(); + + if(isInPlaceReplacement) { + return doInPlaceReplacement(record, writeSchema, flowFile, context, flowFileContext); + } else { + return doResultPathReplacement(record, writeSchema, flowFile, context, flowFileContext); + } + + } + + private Set<Relationship> doInPlaceReplacement(Record record, RecordSchema writeSchema, FlowFile flowFile, + ProcessContext context, Tuple<Map<String, RecordPath>, RecordPath> flowFileContext) { + + final String lookupKey = (String) context.getProperty(LOOKUP_SERVICE).asControllerService(LookupService.class).getRequiredKeys().iterator().next(); + + final Map<String, RecordPath> recordPaths = flowFileContext.getKey(); + final Map<String, Object> lookupCoordinates = new HashMap<>(recordPaths.size()); + + for (final Map.Entry<String, RecordPath> entry : recordPaths.entrySet()) { + final String coordinateKey = entry.getKey(); + final RecordPath recordPath = entry.getValue(); + + final RecordPathResult pathResult = recordPath.evaluate(record); + final List<FieldValue> lookupFieldValues = pathResult.getSelectedFields() + .filter(fieldVal -> fieldVal.getValue() != null) + .collect(Collectors.toList()); + + if (lookupFieldValues.isEmpty()) { + final Set<Relationship> rels = routeToMatchedUnmatched ? UNMATCHED_COLLECTION : SUCCESS_COLLECTION; + getLogger().debug("RecordPath for property '{}' did not match any fields in a record for {}; routing record to {}", new Object[] {coordinateKey, flowFile, rels}); + return rels; + } + + for (FieldValue fieldValue : lookupFieldValues) { + final Object coordinateValue = (fieldValue.getValue() instanceof Number || fieldValue.getValue() instanceof Boolean) + ? fieldValue.getValue() : DataTypeUtils.toString(fieldValue.getValue(), (String) null); + lookupCoordinates.put(lookupKey, coordinateValue); + + final Optional<?> lookupValueOption; + try { + lookupValueOption = lookupService.lookup(lookupCoordinates, flowFile.getAttributes()); + } catch (final Exception e) { + throw new ProcessException("Failed to lookup coordinates " + lookupCoordinates + " in Lookup Service", e); + } + + if (!lookupValueOption.isPresent()) { + final Set<Relationship> rels = routeToMatchedUnmatched ? UNMATCHED_COLLECTION : SUCCESS_COLLECTION; + return rels; + } + + final Object lookupValue = lookupValueOption.get(); + + final DataType inferredDataType = DataTypeUtils.inferDataType(lookupValue, RecordFieldType.STRING.getDataType()); + fieldValue.updateValue(lookupValue, inferredDataType); + + } + } + + final Set<Relationship> rels = routeToMatchedUnmatched ? MATCHED_COLLECTION : SUCCESS_COLLECTION; + return rels; + } + + private Set<Relationship> doResultPathReplacement(Record record, RecordSchema writeSchema, FlowFile flowFile, Review comment: `writeSchema` is not used. Can remove it. ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org With regards, Apache Git Services