mirror of https://github.com/apache/nifi.git
Merge branch 'develop' into NIFI-680
Conflicts: nifi/nifi-mock/src/main/java/org/apache/nifi/util/StandardProcessorTestRunner.java nifi/nifi-mock/src/main/java/org/apache/nifi/util/TestRunner.java nifi/nifi-mock/src/test/java/org/apache/nifi/util/CurrentTestStandardProcessorTestRunner.java
This commit is contained in:
commit
cc3be416b2
|
@ -22,6 +22,7 @@ public interface LineageNode {
|
|||
* @return the identifier of the Clustered NiFi Node that generated the
|
||||
* event
|
||||
*/
|
||||
@Deprecated
|
||||
String getClusterNodeIdentifier();
|
||||
|
||||
/**
|
||||
|
|
|
@ -276,6 +276,7 @@ language governing permissions and limitations under the License. -->
|
|||
<nifi.provenance.repository.index.shard.size>500 MB</nifi.provenance.repository.index.shard.size>
|
||||
<nifi.provenance.repository.always.sync>false</nifi.provenance.repository.always.sync>
|
||||
<nifi.provenance.repository.journal.count>16</nifi.provenance.repository.journal.count>
|
||||
<nifi.provenance.repository.max.attribute.length>65536</nifi.provenance.repository.max.attribute.length>
|
||||
|
||||
<!-- volatile provenance repository properties -->
|
||||
<nifi.provenance.repository.buffer.size>100000</nifi.provenance.repository.buffer.size>
|
||||
|
|
|
@ -36,11 +36,13 @@ public class EventNode implements ProvenanceEventLineageNode {
|
|||
return String.valueOf(getEventIdentifier());
|
||||
}
|
||||
|
||||
@Deprecated
|
||||
@Override
|
||||
public String getClusterNodeIdentifier() {
|
||||
return clusterNodeIdentifier;
|
||||
}
|
||||
|
||||
@Deprecated
|
||||
public void setClusterNodeIdentifier(final String nodeIdentifier) {
|
||||
this.clusterNodeIdentifier = nodeIdentifier;
|
||||
}
|
||||
|
|
|
@ -39,6 +39,7 @@ public class FlowFileNode implements LineageNode {
|
|||
return creationTime;
|
||||
}
|
||||
|
||||
@Deprecated
|
||||
@Override
|
||||
public String getClusterNodeIdentifier() {
|
||||
return clusterNodeIdentifier;
|
||||
|
|
|
@ -150,6 +150,7 @@ TO_RADIX : 'toRadix';
|
|||
OR : 'or';
|
||||
AND : 'and';
|
||||
JOIN : 'join';
|
||||
TO_LITERAL : 'literal';
|
||||
|
||||
// 2 arg functions
|
||||
SUBSTRING : 'substring';
|
||||
|
|
|
@ -127,7 +127,8 @@ functionCall : functionRef ->
|
|||
|
||||
booleanLiteral : TRUE | FALSE;
|
||||
zeroArgStandaloneFunction : (IP | UUID | NOW | NEXT_INT | HOSTNAME) LPAREN! RPAREN!;
|
||||
oneArgStandaloneFunction : HOSTNAME^ LPAREN! booleanLiteral RPAREN!;
|
||||
oneArgStandaloneFunction : (TO_LITERAL^ LPAREN! anyArg RPAREN!) |
|
||||
(HOSTNAME^ LPAREN! booleanLiteral RPAREN!);
|
||||
standaloneFunction : zeroArgStandaloneFunction | oneArgStandaloneFunction;
|
||||
|
||||
attributeRefOrFunctionCall : (attributeRef | standaloneFunction);
|
||||
|
|
|
@ -95,12 +95,14 @@ import org.apache.nifi.attribute.expression.language.evaluation.functions.UuidEv
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.literals.BooleanLiteralEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.literals.NumberLiteralEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.literals.StringLiteralEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.literals.ToLiteralEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.reduce.CountEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.reduce.JoinEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.reduce.ReduceEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.AllAttributesEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.AnyAttributeEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.DelineatedAttributeEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.IteratingEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.MultiAttributeEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.MultiMatchAttributeEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.MultiNamedAttributeEvaluator;
|
||||
|
@ -114,6 +116,7 @@ import org.antlr.runtime.ANTLRStringStream;
|
|||
import org.antlr.runtime.CharStream;
|
||||
import org.antlr.runtime.CommonTokenStream;
|
||||
import org.antlr.runtime.tree.Tree;
|
||||
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.ALL_ATTRIBUTES;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.ALL_DELINEATED_VALUES;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.ALL_MATCHING_ATTRIBUTES;
|
||||
|
@ -171,6 +174,7 @@ import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpre
|
|||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.SUBSTRING_BEFORE;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.SUBSTRING_BEFORE_LAST;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.TO_DATE;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.TO_LITERAL;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.TO_LOWER;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.TO_NUMBER;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.TO_RADIX;
|
||||
|
@ -181,6 +185,7 @@ import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpre
|
|||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.URL_DECODE;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.URL_ENCODE;
|
||||
import static org.apache.nifi.attribute.expression.language.antlr.AttributeExpressionParser.UUID;
|
||||
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.selection.MappingEvaluator;
|
||||
|
||||
/**
|
||||
|
@ -205,7 +210,7 @@ public class Query {
|
|||
try {
|
||||
validateExpression(value, false);
|
||||
return true;
|
||||
} catch (final ProcessException e) {
|
||||
} catch (final AttributeExpressionLanguageParsingException | ProcessException e) {
|
||||
return false;
|
||||
}
|
||||
}
|
||||
|
@ -341,7 +346,7 @@ public class Query {
|
|||
|
||||
if (c == '\\') {
|
||||
backslashCount++;
|
||||
} else if (c == quoteChar && ((backslashCount % 2 == 0) || lastChar != '\\')) {
|
||||
} else if (c == quoteChar && (backslashCount % 2 == 0 || lastChar != '\\')) {
|
||||
return i;
|
||||
}
|
||||
|
||||
|
@ -359,7 +364,7 @@ public class Query {
|
|||
|
||||
final String value = evaluated.toString();
|
||||
final String escaped = value.replace("$$", "$");
|
||||
return (decorator == null) ? escaped : decorator.decorate(escaped);
|
||||
return decorator == null ? escaped : decorator.decorate(escaped);
|
||||
}
|
||||
|
||||
static String evaluateExpressions(final String rawValue, Map<String, String> expressionMap) throws ProcessException {
|
||||
|
@ -631,6 +636,25 @@ public class Query {
|
|||
}
|
||||
|
||||
private static void verifyMappingEvaluatorReduced(final Evaluator<?> evaluator) {
|
||||
final Evaluator<?> rightMostEvaluator;
|
||||
if (evaluator instanceof IteratingEvaluator) {
|
||||
rightMostEvaluator = ((IteratingEvaluator<?>) evaluator).getLogicEvaluator();
|
||||
} else {
|
||||
rightMostEvaluator = evaluator;
|
||||
}
|
||||
|
||||
Evaluator<?> eval = rightMostEvaluator.getSubjectEvaluator();
|
||||
Evaluator<?> lastEval = rightMostEvaluator;
|
||||
while (eval != null) {
|
||||
if (eval instanceof ReduceEvaluator) {
|
||||
throw new AttributeExpressionLanguageParsingException("Expression attempts to call function '" + lastEval.getToken() + "' on the result of '" + eval.getToken() +
|
||||
"'. This is not allowed. Instead, use \"${literal( ${<embedded expression>} ):" + lastEval.getToken() + "(...)}\"");
|
||||
}
|
||||
|
||||
lastEval = eval;
|
||||
eval = eval.getSubjectEvaluator();
|
||||
}
|
||||
|
||||
// if the result type of the evaluator is BOOLEAN, then it will always
|
||||
// be reduced when evaluator.
|
||||
final ResultType resultType = evaluator.getResultType();
|
||||
|
@ -691,7 +715,7 @@ public class Query {
|
|||
return "Query [" + query + "]";
|
||||
}
|
||||
|
||||
private static StringEvaluator newStringLiteralEvaluator(final String literalValue) {
|
||||
private static Evaluator<String> newStringLiteralEvaluator(final String literalValue) {
|
||||
if (literalValue == null || literalValue.length() < 2) {
|
||||
return new StringLiteralEvaluator(literalValue);
|
||||
}
|
||||
|
@ -724,7 +748,7 @@ public class Query {
|
|||
return toStringEvaluator(evaluators.get(0));
|
||||
}
|
||||
|
||||
StringEvaluator lastEvaluator = toStringEvaluator(evaluators.get(0));
|
||||
Evaluator<String> lastEvaluator = toStringEvaluator(evaluators.get(0));
|
||||
for (int i = 1; i < evaluators.size(); i++) {
|
||||
lastEvaluator = new AppendEvaluator(lastEvaluator, toStringEvaluator(evaluators.get(i)));
|
||||
}
|
||||
|
@ -749,8 +773,8 @@ public class Query {
|
|||
final Tree functionTypeTree = tree.getChild(0);
|
||||
final int multiAttrType = functionTypeTree.getType();
|
||||
if (multiAttrType == ANY_DELINEATED_VALUE || multiAttrType == ALL_DELINEATED_VALUES) {
|
||||
final StringEvaluator delineatedValueEvaluator = toStringEvaluator(buildEvaluator(tree.getChild(1)));
|
||||
final StringEvaluator delimiterEvaluator = toStringEvaluator(buildEvaluator(tree.getChild(2)));
|
||||
final Evaluator<String> delineatedValueEvaluator = toStringEvaluator(buildEvaluator(tree.getChild(1)));
|
||||
final Evaluator<String> delimiterEvaluator = toStringEvaluator(buildEvaluator(tree.getChild(2)));
|
||||
|
||||
return new DelineatedAttributeEvaluator(delineatedValueEvaluator, delimiterEvaluator, multiAttrType);
|
||||
}
|
||||
|
@ -807,6 +831,10 @@ public class Query {
|
|||
case NOW: {
|
||||
return new NowEvaluator();
|
||||
}
|
||||
case TO_LITERAL: {
|
||||
final Evaluator<?> argEvaluator = buildEvaluator(tree.getChild(0));
|
||||
return new ToLiteralEvaluator(argEvaluator);
|
||||
}
|
||||
case IP: {
|
||||
try {
|
||||
return new IPEvaluator();
|
||||
|
@ -818,7 +846,7 @@ public class Query {
|
|||
if (tree.getChildCount() == 0) {
|
||||
try {
|
||||
return new HostnameEvaluator(false);
|
||||
} catch (UnknownHostException e) {
|
||||
} catch (final UnknownHostException e) {
|
||||
throw new AttributeExpressionLanguageException(e);
|
||||
}
|
||||
} else if (tree.getChildCount() == 1) {
|
||||
|
@ -832,7 +860,7 @@ public class Query {
|
|||
default:
|
||||
throw new AttributeExpressionLanguageParsingException("Call to hostname() must take 0 or 1 (boolean) parameter");
|
||||
}
|
||||
} catch (UnknownHostException e) {
|
||||
} catch (final UnknownHostException e) {
|
||||
throw new AttributeExpressionLanguageException(e);
|
||||
}
|
||||
} else {
|
||||
|
@ -847,16 +875,22 @@ public class Query {
|
|||
}
|
||||
}
|
||||
|
||||
private static <T> Evaluator<T> addToken(final Evaluator<T> evaluator, final String token) {
|
||||
evaluator.setToken(token);
|
||||
return evaluator;
|
||||
}
|
||||
|
||||
private static Evaluator<Boolean> buildBooleanEvaluator(final Tree tree) {
|
||||
switch (tree.getType()) {
|
||||
case TRUE:
|
||||
return new BooleanLiteralEvaluator(true);
|
||||
return addToken(new BooleanLiteralEvaluator(true), "true");
|
||||
case FALSE:
|
||||
return new BooleanLiteralEvaluator(false);
|
||||
return addToken(new BooleanLiteralEvaluator(false), "true");
|
||||
}
|
||||
throw new AttributeExpressionLanguageParsingException("Cannot build Boolean evaluator from tree " + tree.toString());
|
||||
}
|
||||
|
||||
@SuppressWarnings({ "rawtypes", "unchecked" })
|
||||
private static Evaluator<?> buildExpressionEvaluator(final Tree tree) {
|
||||
if (tree.getChildCount() == 0) {
|
||||
throw new AttributeExpressionLanguageParsingException("EXPRESSION tree node has no children");
|
||||
|
@ -901,6 +935,27 @@ public class Query {
|
|||
break;
|
||||
}
|
||||
}
|
||||
|
||||
switch (multiAttrEval.getEvaluationType()) {
|
||||
case ANY_ATTRIBUTE:
|
||||
chosenEvaluator.setToken("anyAttribute");
|
||||
break;
|
||||
case ANY_MATCHING_ATTRIBUTE:
|
||||
chosenEvaluator.setToken("anyMatchingAttribute");
|
||||
break;
|
||||
case ANY_DELINEATED_VALUE:
|
||||
chosenEvaluator.setToken("anyDelineatedValue");
|
||||
break;
|
||||
case ALL_ATTRIBUTES:
|
||||
chosenEvaluator.setToken("allAttributes");
|
||||
break;
|
||||
case ALL_MATCHING_ATTRIBUTES:
|
||||
chosenEvaluator.setToken("allMatchingAttributes");
|
||||
break;
|
||||
case ALL_DELINEATED_VALUES:
|
||||
chosenEvaluator.setToken("allDelineatedValues");
|
||||
break;
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
|
@ -934,24 +989,25 @@ public class Query {
|
|||
return args;
|
||||
}
|
||||
|
||||
private static StringEvaluator toStringEvaluator(final Evaluator<?> evaluator) {
|
||||
private static Evaluator<String> toStringEvaluator(final Evaluator<?> evaluator) {
|
||||
return toStringEvaluator(evaluator, null);
|
||||
}
|
||||
|
||||
private static StringEvaluator toStringEvaluator(final Evaluator<?> evaluator, final String location) {
|
||||
private static Evaluator<String> toStringEvaluator(final Evaluator<?> evaluator, final String location) {
|
||||
if (evaluator.getResultType() == ResultType.STRING) {
|
||||
return (StringEvaluator) evaluator;
|
||||
}
|
||||
|
||||
return new StringCastEvaluator(evaluator);
|
||||
return addToken(new StringCastEvaluator(evaluator), evaluator.getToken());
|
||||
}
|
||||
|
||||
private static BooleanEvaluator toBooleanEvaluator(final Evaluator<?> evaluator, final String location) {
|
||||
@SuppressWarnings("unchecked")
|
||||
private static Evaluator<Boolean> toBooleanEvaluator(final Evaluator<?> evaluator, final String location) {
|
||||
switch (evaluator.getResultType()) {
|
||||
case BOOLEAN:
|
||||
return (BooleanEvaluator) evaluator;
|
||||
return (Evaluator<Boolean>) evaluator;
|
||||
case STRING:
|
||||
return new BooleanCastEvaluator((StringEvaluator) evaluator);
|
||||
return addToken(new BooleanCastEvaluator((StringEvaluator) evaluator), evaluator.getToken());
|
||||
default:
|
||||
throw new AttributeExpressionLanguageParsingException("Cannot implicitly convert Data Type " + evaluator.getResultType() + " to " + ResultType.BOOLEAN
|
||||
+ (location == null ? "" : " at location [" + location + "]"));
|
||||
|
@ -959,22 +1015,23 @@ public class Query {
|
|||
|
||||
}
|
||||
|
||||
private static BooleanEvaluator toBooleanEvaluator(final Evaluator<?> evaluator) {
|
||||
private static Evaluator<Boolean> toBooleanEvaluator(final Evaluator<?> evaluator) {
|
||||
return toBooleanEvaluator(evaluator, null);
|
||||
}
|
||||
|
||||
private static NumberEvaluator toNumberEvaluator(final Evaluator<?> evaluator) {
|
||||
private static Evaluator<Long> toNumberEvaluator(final Evaluator<?> evaluator) {
|
||||
return toNumberEvaluator(evaluator, null);
|
||||
}
|
||||
|
||||
private static NumberEvaluator toNumberEvaluator(final Evaluator<?> evaluator, final String location) {
|
||||
@SuppressWarnings("unchecked")
|
||||
private static Evaluator<Long> toNumberEvaluator(final Evaluator<?> evaluator, final String location) {
|
||||
switch (evaluator.getResultType()) {
|
||||
case NUMBER:
|
||||
return (NumberEvaluator) evaluator;
|
||||
return (Evaluator<Long>) evaluator;
|
||||
case STRING:
|
||||
return new NumberCastEvaluator(evaluator);
|
||||
return addToken(new NumberCastEvaluator(evaluator), evaluator.getToken());
|
||||
case DATE:
|
||||
return new DateToNumberEvaluator((DateEvaluator) evaluator);
|
||||
return addToken(new DateToNumberEvaluator((DateEvaluator) evaluator), evaluator.getToken());
|
||||
default:
|
||||
throw new AttributeExpressionLanguageParsingException("Cannot implicitly convert Data Type " + evaluator.getResultType() + " to " + ResultType.NUMBER
|
||||
+ (location == null ? "" : " at location [" + location + "]"));
|
||||
|
@ -997,233 +1054,233 @@ public class Query {
|
|||
switch (tree.getType()) {
|
||||
case TRIM: {
|
||||
verifyArgCount(argEvaluators, 0, "trim");
|
||||
return new TrimEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new TrimEvaluator(toStringEvaluator(subjectEvaluator)), "trim");
|
||||
}
|
||||
case TO_STRING: {
|
||||
verifyArgCount(argEvaluators, 0, "toString");
|
||||
return new ToStringEvaluator(subjectEvaluator);
|
||||
return addToken(new ToStringEvaluator(subjectEvaluator), "toString");
|
||||
}
|
||||
case TO_LOWER: {
|
||||
verifyArgCount(argEvaluators, 0, "toLower");
|
||||
return new ToLowerEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new ToLowerEvaluator(toStringEvaluator(subjectEvaluator)), "toLower");
|
||||
}
|
||||
case TO_UPPER: {
|
||||
verifyArgCount(argEvaluators, 0, "toUpper");
|
||||
return new ToUpperEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new ToUpperEvaluator(toStringEvaluator(subjectEvaluator)), "toUpper");
|
||||
}
|
||||
case URL_ENCODE: {
|
||||
verifyArgCount(argEvaluators, 0, "urlEncode");
|
||||
return new UrlEncodeEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new UrlEncodeEvaluator(toStringEvaluator(subjectEvaluator)), "urlEncode");
|
||||
}
|
||||
case URL_DECODE: {
|
||||
verifyArgCount(argEvaluators, 0, "urlDecode");
|
||||
return new UrlDecodeEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new UrlDecodeEvaluator(toStringEvaluator(subjectEvaluator)), "urlDecode");
|
||||
}
|
||||
case SUBSTRING_BEFORE: {
|
||||
verifyArgCount(argEvaluators, 1, "substringBefore");
|
||||
return new SubstringBeforeEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringBefore"));
|
||||
return addToken(new SubstringBeforeEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringBefore")), "substringBefore");
|
||||
}
|
||||
case SUBSTRING_BEFORE_LAST: {
|
||||
verifyArgCount(argEvaluators, 1, "substringBeforeLast");
|
||||
return new SubstringBeforeLastEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringBeforeLast"));
|
||||
return addToken(new SubstringBeforeLastEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringBeforeLast")), "substringBeforeLast");
|
||||
}
|
||||
case SUBSTRING_AFTER: {
|
||||
verifyArgCount(argEvaluators, 1, "substringAfter");
|
||||
return new SubstringAfterEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringAfter"));
|
||||
return addToken(new SubstringAfterEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringAfter")), "substringAfter");
|
||||
}
|
||||
case SUBSTRING_AFTER_LAST: {
|
||||
verifyArgCount(argEvaluators, 1, "substringAfterLast");
|
||||
return new SubstringAfterLastEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringAfterLast"));
|
||||
return addToken(new SubstringAfterLastEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to substringAfterLast")), "substringAfterLast");
|
||||
}
|
||||
case REPLACE_NULL: {
|
||||
verifyArgCount(argEvaluators, 1, "replaceNull");
|
||||
return new ReplaceNullEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to replaceNull"));
|
||||
return addToken(new ReplaceNullEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to replaceNull")), "replaceNull");
|
||||
}
|
||||
case REPLACE_EMPTY: {
|
||||
verifyArgCount(argEvaluators, 1, "replaceEmtpy");
|
||||
return new ReplaceEmptyEvaluator(toStringEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0), "first argumen to replaceEmpty"));
|
||||
return addToken(new ReplaceEmptyEvaluator(toStringEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0), "first argument to replaceEmpty")), "replaceEmpty");
|
||||
}
|
||||
case REPLACE: {
|
||||
verifyArgCount(argEvaluators, 2, "replace");
|
||||
return new ReplaceEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
return addToken(new ReplaceEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to replace"),
|
||||
toStringEvaluator(argEvaluators.get(1), "second argument to replace"));
|
||||
toStringEvaluator(argEvaluators.get(1), "second argument to replace")), "replace");
|
||||
}
|
||||
case REPLACE_ALL: {
|
||||
verifyArgCount(argEvaluators, 2, "replaceAll");
|
||||
return new ReplaceAllEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
return addToken(new ReplaceAllEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to replaceAll"),
|
||||
toStringEvaluator(argEvaluators.get(1), "second argument to replaceAll"));
|
||||
toStringEvaluator(argEvaluators.get(1), "second argument to replaceAll")), "replaceAll");
|
||||
}
|
||||
case APPEND: {
|
||||
verifyArgCount(argEvaluators, 1, "append");
|
||||
return new AppendEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to append"));
|
||||
return addToken(new AppendEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to append")), "append");
|
||||
}
|
||||
case PREPEND: {
|
||||
verifyArgCount(argEvaluators, 1, "prepend");
|
||||
return new PrependEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to prepend"));
|
||||
return addToken(new PrependEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to prepend")), "prepend");
|
||||
}
|
||||
case SUBSTRING: {
|
||||
final int numArgs = argEvaluators.size();
|
||||
if (numArgs == 1) {
|
||||
return new SubstringEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to substring"));
|
||||
return addToken(new SubstringEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to substring")), "substring");
|
||||
} else if (numArgs == 2) {
|
||||
return new SubstringEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
return addToken(new SubstringEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to substring"),
|
||||
toNumberEvaluator(argEvaluators.get(1), "second argument to substring"));
|
||||
toNumberEvaluator(argEvaluators.get(1), "second argument to substring")), "substring");
|
||||
} else {
|
||||
throw new AttributeExpressionLanguageParsingException("substring() function can take either 1 or 2 arguments but cannot take " + numArgs + " arguments");
|
||||
}
|
||||
}
|
||||
case JOIN: {
|
||||
verifyArgCount(argEvaluators, 1, "join");
|
||||
return new JoinEvaluator(toStringEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new JoinEvaluator(toStringEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0))), "join");
|
||||
}
|
||||
case COUNT: {
|
||||
verifyArgCount(argEvaluators, 0, "count");
|
||||
return new CountEvaluator(subjectEvaluator);
|
||||
return addToken(new CountEvaluator(subjectEvaluator), "count");
|
||||
}
|
||||
case IS_NULL: {
|
||||
verifyArgCount(argEvaluators, 0, "isNull");
|
||||
return new IsNullEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new IsNullEvaluator(toStringEvaluator(subjectEvaluator)), "isNull");
|
||||
}
|
||||
case IS_EMPTY: {
|
||||
verifyArgCount(argEvaluators, 0, "isNull");
|
||||
return new IsEmptyEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new IsEmptyEvaluator(toStringEvaluator(subjectEvaluator)), "isEmpty");
|
||||
}
|
||||
case NOT_NULL: {
|
||||
verifyArgCount(argEvaluators, 0, "notNull");
|
||||
return new NotNullEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new NotNullEvaluator(toStringEvaluator(subjectEvaluator)), "notNull");
|
||||
}
|
||||
case STARTS_WITH: {
|
||||
verifyArgCount(argEvaluators, 1, "startsWith");
|
||||
return new StartsWithEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to startsWith"));
|
||||
return addToken(new StartsWithEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to startsWith")), "startsWith");
|
||||
}
|
||||
case ENDS_WITH: {
|
||||
verifyArgCount(argEvaluators, 1, "endsWith");
|
||||
return new EndsWithEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to endsWith"));
|
||||
return addToken(new EndsWithEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to endsWith")), "endsWith");
|
||||
}
|
||||
case CONTAINS: {
|
||||
verifyArgCount(argEvaluators, 1, "contains");
|
||||
return new ContainsEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to contains"));
|
||||
return addToken(new ContainsEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to contains")), "contains");
|
||||
}
|
||||
case FIND: {
|
||||
verifyArgCount(argEvaluators, 1, "find");
|
||||
return new FindEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to find"));
|
||||
return addToken(new FindEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to find")), "find");
|
||||
}
|
||||
case MATCHES: {
|
||||
verifyArgCount(argEvaluators, 1, "matches");
|
||||
return new MatchesEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to matches"));
|
||||
return addToken(new MatchesEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to matches")), "matches");
|
||||
}
|
||||
case EQUALS: {
|
||||
verifyArgCount(argEvaluators, 1, "equals");
|
||||
return new EqualsEvaluator(subjectEvaluator, argEvaluators.get(0));
|
||||
return addToken(new EqualsEvaluator(subjectEvaluator, argEvaluators.get(0)), "equals");
|
||||
}
|
||||
case EQUALS_IGNORE_CASE: {
|
||||
verifyArgCount(argEvaluators, 1, "equalsIgnoreCase");
|
||||
return new EqualsIgnoreCaseEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to equalsIgnoreCase"));
|
||||
return addToken(new EqualsIgnoreCaseEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to equalsIgnoreCase")), "equalsIgnoreCase");
|
||||
}
|
||||
case GREATER_THAN: {
|
||||
verifyArgCount(argEvaluators, 1, "gt");
|
||||
return new GreaterThanEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to gt"));
|
||||
return addToken(new GreaterThanEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to gt")), "gt");
|
||||
}
|
||||
case GREATER_THAN_OR_EQUAL: {
|
||||
verifyArgCount(argEvaluators, 1, "ge");
|
||||
return new GreaterThanOrEqualEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to ge"));
|
||||
return addToken(new GreaterThanOrEqualEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to ge")), "ge");
|
||||
}
|
||||
case LESS_THAN: {
|
||||
verifyArgCount(argEvaluators, 1, "lt");
|
||||
return new LessThanEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to lt"));
|
||||
return addToken(new LessThanEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to lt")), "lt");
|
||||
}
|
||||
case LESS_THAN_OR_EQUAL: {
|
||||
verifyArgCount(argEvaluators, 1, "le");
|
||||
return new LessThanOrEqualEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to le"));
|
||||
return addToken(new LessThanOrEqualEvaluator(toNumberEvaluator(subjectEvaluator),
|
||||
toNumberEvaluator(argEvaluators.get(0), "first argument to le")), "le");
|
||||
}
|
||||
case LENGTH: {
|
||||
verifyArgCount(argEvaluators, 0, "length");
|
||||
return new LengthEvaluator(toStringEvaluator(subjectEvaluator));
|
||||
return addToken(new LengthEvaluator(toStringEvaluator(subjectEvaluator)), "length");
|
||||
}
|
||||
case TO_DATE: {
|
||||
if (argEvaluators.isEmpty()) {
|
||||
return new NumberToDateEvaluator(toNumberEvaluator(subjectEvaluator));
|
||||
return addToken(new NumberToDateEvaluator(toNumberEvaluator(subjectEvaluator)), "toDate");
|
||||
} else if (subjectEvaluator.getResultType() == ResultType.STRING) {
|
||||
return new StringToDateEvaluator(toStringEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new StringToDateEvaluator(toStringEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0))), "toDate");
|
||||
} else {
|
||||
return new NumberToDateEvaluator(toNumberEvaluator(subjectEvaluator));
|
||||
return addToken(new NumberToDateEvaluator(toNumberEvaluator(subjectEvaluator)), "toDate");
|
||||
}
|
||||
}
|
||||
case TO_NUMBER: {
|
||||
verifyArgCount(argEvaluators, 0, "toNumber");
|
||||
switch (subjectEvaluator.getResultType()) {
|
||||
case STRING:
|
||||
return new ToNumberEvaluator((StringEvaluator) subjectEvaluator);
|
||||
return addToken(new ToNumberEvaluator((StringEvaluator) subjectEvaluator), "toNumber");
|
||||
case DATE:
|
||||
return new DateToNumberEvaluator((DateEvaluator) subjectEvaluator);
|
||||
return addToken(new DateToNumberEvaluator((DateEvaluator) subjectEvaluator), "toNumber");
|
||||
default:
|
||||
throw new AttributeExpressionLanguageParsingException(subjectEvaluator + " returns type " + subjectEvaluator.getResultType() + " but expected to get " + ResultType.STRING);
|
||||
}
|
||||
}
|
||||
case TO_RADIX: {
|
||||
if (argEvaluators.size() == 1) {
|
||||
return new ToRadixEvaluator((NumberEvaluator) subjectEvaluator, toNumberEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new ToRadixEvaluator((NumberEvaluator) subjectEvaluator, toNumberEvaluator(argEvaluators.get(0))), "toRadix");
|
||||
} else {
|
||||
return new ToRadixEvaluator((NumberEvaluator) subjectEvaluator, toNumberEvaluator(argEvaluators.get(0)), toNumberEvaluator(argEvaluators.get(1)));
|
||||
return addToken(new ToRadixEvaluator((NumberEvaluator) subjectEvaluator, toNumberEvaluator(argEvaluators.get(0)), toNumberEvaluator(argEvaluators.get(1))), "toRadix");
|
||||
}
|
||||
}
|
||||
case MOD: {
|
||||
return new ModEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new ModEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0))), "mod");
|
||||
}
|
||||
case PLUS: {
|
||||
return new PlusEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new PlusEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0))), "plus");
|
||||
}
|
||||
case MINUS: {
|
||||
return new MinusEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new MinusEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0))), "minus");
|
||||
}
|
||||
case MULTIPLY: {
|
||||
return new MultiplyEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new MultiplyEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0))), "multiply");
|
||||
}
|
||||
case DIVIDE: {
|
||||
return new DivideEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new DivideEvaluator(toNumberEvaluator(subjectEvaluator), toNumberEvaluator(argEvaluators.get(0))), "divide");
|
||||
}
|
||||
case INDEX_OF: {
|
||||
verifyArgCount(argEvaluators, 1, "indexOf");
|
||||
return new IndexOfEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to indexOf"));
|
||||
return addToken(new IndexOfEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to indexOf")), "indexOf");
|
||||
}
|
||||
case LAST_INDEX_OF: {
|
||||
verifyArgCount(argEvaluators, 1, "lastIndexOf");
|
||||
return new LastIndexOfEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to lastIndexOf"));
|
||||
return addToken(new LastIndexOfEvaluator(toStringEvaluator(subjectEvaluator),
|
||||
toStringEvaluator(argEvaluators.get(0), "first argument to lastIndexOf")), "lastIndexOf");
|
||||
}
|
||||
case FORMAT: {
|
||||
return new FormatEvaluator(toDateEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0), "first argument of format"));
|
||||
return addToken(new FormatEvaluator(toDateEvaluator(subjectEvaluator), toStringEvaluator(argEvaluators.get(0), "first argument of format")), "format");
|
||||
}
|
||||
case OR: {
|
||||
return new OrEvaluator(toBooleanEvaluator(subjectEvaluator), toBooleanEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new OrEvaluator(toBooleanEvaluator(subjectEvaluator), toBooleanEvaluator(argEvaluators.get(0))), "or");
|
||||
}
|
||||
case AND: {
|
||||
return new AndEvaluator(toBooleanEvaluator(subjectEvaluator), toBooleanEvaluator(argEvaluators.get(0)));
|
||||
return addToken(new AndEvaluator(toBooleanEvaluator(subjectEvaluator), toBooleanEvaluator(argEvaluators.get(0))), "and");
|
||||
}
|
||||
case NOT: {
|
||||
return new NotEvaluator(toBooleanEvaluator(subjectEvaluator));
|
||||
return addToken(new NotEvaluator(toBooleanEvaluator(subjectEvaluator)), "not");
|
||||
}
|
||||
default:
|
||||
throw new AttributeExpressionLanguageParsingException("Expected a Function-type expression but got " + tree.toString());
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.nifi.attribute.expression.language.evaluation;
|
|||
import org.apache.nifi.expression.AttributeExpression.ResultType;
|
||||
|
||||
public abstract class BooleanEvaluator implements Evaluator<Boolean> {
|
||||
private String token;
|
||||
|
||||
@Override
|
||||
public ResultType getResultType() {
|
||||
|
@ -29,4 +30,14 @@ public abstract class BooleanEvaluator implements Evaluator<Boolean> {
|
|||
public int getEvaluationsRemaining() {
|
||||
return 0;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String getToken() {
|
||||
return token;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void setToken(final String token) {
|
||||
this.token = token;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -21,6 +21,7 @@ import java.util.Date;
|
|||
import org.apache.nifi.expression.AttributeExpression.ResultType;
|
||||
|
||||
public abstract class DateEvaluator implements Evaluator<Date> {
|
||||
private String token;
|
||||
|
||||
@Override
|
||||
public ResultType getResultType() {
|
||||
|
@ -31,4 +32,14 @@ public abstract class DateEvaluator implements Evaluator<Date> {
|
|||
public int getEvaluationsRemaining() {
|
||||
return 0;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String getToken() {
|
||||
return token;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void setToken(final String token) {
|
||||
this.token = token;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -29,4 +29,16 @@ public interface Evaluator<T> {
|
|||
int getEvaluationsRemaining();
|
||||
|
||||
Evaluator<?> getSubjectEvaluator();
|
||||
|
||||
/**
|
||||
* Sets the token that was used in the query to cause this Evaluator to be created
|
||||
*
|
||||
* @param token the token that caused this Evaluator to be created
|
||||
*/
|
||||
void setToken(String token);
|
||||
|
||||
/**
|
||||
* @return the token that caused this Evaluator to be created
|
||||
*/
|
||||
String getToken();
|
||||
}
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.nifi.attribute.expression.language.evaluation;
|
|||
import org.apache.nifi.expression.AttributeExpression.ResultType;
|
||||
|
||||
public abstract class NumberEvaluator implements Evaluator<Long> {
|
||||
private String token;
|
||||
|
||||
@Override
|
||||
public ResultType getResultType() {
|
||||
|
@ -30,4 +31,13 @@ public abstract class NumberEvaluator implements Evaluator<Long> {
|
|||
return 0;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String getToken() {
|
||||
return token;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void setToken(final String token) {
|
||||
this.token = token;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -19,6 +19,7 @@ package org.apache.nifi.attribute.expression.language.evaluation;
|
|||
import org.apache.nifi.expression.AttributeExpression.ResultType;
|
||||
|
||||
public abstract class StringEvaluator implements Evaluator<String> {
|
||||
private String token;
|
||||
|
||||
@Override
|
||||
public ResultType getResultType() {
|
||||
|
@ -29,4 +30,14 @@ public abstract class StringEvaluator implements Evaluator<String> {
|
|||
public int getEvaluationsRemaining() {
|
||||
return 0;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String getToken() {
|
||||
return token;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void setToken(final String token) {
|
||||
this.token = token;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class AndEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final BooleanEvaluator subjectEvaluator;
|
||||
private final BooleanEvaluator rhsEvaluator;
|
||||
private final Evaluator<Boolean> subjectEvaluator;
|
||||
private final Evaluator<Boolean> rhsEvaluator;
|
||||
|
||||
public AndEvaluator(final BooleanEvaluator subjectEvaluator, final BooleanEvaluator rhsEvaluator) {
|
||||
public AndEvaluator(final Evaluator<Boolean> subjectEvaluator, final Evaluator<Boolean> rhsEvaluator) {
|
||||
this.subjectEvaluator = subjectEvaluator;
|
||||
this.rhsEvaluator = rhsEvaluator;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class AppendEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator appendEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> appendEvaluator;
|
||||
|
||||
public AppendEvaluator(final StringEvaluator subject, final StringEvaluator append) {
|
||||
public AppendEvaluator(final Evaluator<String> subject, final Evaluator<String> append) {
|
||||
this.subject = subject;
|
||||
this.appendEvaluator = append;
|
||||
}
|
||||
|
|
|
@ -25,9 +25,9 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class AttributeEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator nameEvaluator;
|
||||
private final Evaluator<String> nameEvaluator;
|
||||
|
||||
public AttributeEvaluator(final StringEvaluator nameEvaluator) {
|
||||
public AttributeEvaluator(final Evaluator<String> nameEvaluator) {
|
||||
this.nameEvaluator = nameEvaluator;
|
||||
}
|
||||
|
||||
|
|
|
@ -22,14 +22,13 @@ import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public class ContainsEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator search;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> search;
|
||||
|
||||
public ContainsEvaluator(final StringEvaluator subject, final StringEvaluator search) {
|
||||
public ContainsEvaluator(final Evaluator<String> subject, final Evaluator<String> search) {
|
||||
this.subject = subject;
|
||||
this.search = search;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class DivideEvaluator extends NumberEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator divideValue;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> divideValue;
|
||||
|
||||
public DivideEvaluator(final NumberEvaluator subject, final NumberEvaluator divideValue) {
|
||||
public DivideEvaluator(final Evaluator<Long> subject, final Evaluator<Long> divideValue) {
|
||||
this.subject = subject;
|
||||
this.divideValue = divideValue;
|
||||
}
|
||||
|
|
|
@ -22,14 +22,13 @@ import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public class EndsWithEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator search;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> search;
|
||||
|
||||
public EndsWithEvaluator(final StringEvaluator subject, final StringEvaluator search) {
|
||||
public EndsWithEvaluator(final Evaluator<String> subject, final Evaluator<String> search) {
|
||||
this.subject = subject;
|
||||
this.search = search;
|
||||
}
|
||||
|
|
|
@ -23,17 +23,16 @@ import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.literals.StringLiteralEvaluator;
|
||||
|
||||
public class FindEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator search;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> search;
|
||||
|
||||
private final Pattern compiledPattern;
|
||||
|
||||
public FindEvaluator(final StringEvaluator subject, final StringEvaluator search) {
|
||||
public FindEvaluator(final Evaluator<String> subject, final Evaluator<String> search) {
|
||||
this.subject = subject;
|
||||
this.search = search;
|
||||
|
||||
|
|
|
@ -30,9 +30,9 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
public class FormatEvaluator extends StringEvaluator {
|
||||
|
||||
private final DateEvaluator subject;
|
||||
private final StringEvaluator format;
|
||||
private final Evaluator<String> format;
|
||||
|
||||
public FormatEvaluator(final DateEvaluator subject, final StringEvaluator format) {
|
||||
public FormatEvaluator(final DateEvaluator subject, final Evaluator<String> format) {
|
||||
this.subject = subject;
|
||||
this.format = format;
|
||||
}
|
||||
|
|
|
@ -21,15 +21,14 @@ import java.util.Map;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
|
||||
public class GreaterThanEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator comparison;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> comparison;
|
||||
|
||||
public GreaterThanEvaluator(final NumberEvaluator subject, final NumberEvaluator comparison) {
|
||||
public GreaterThanEvaluator(final Evaluator<Long> subject, final Evaluator<Long> comparison) {
|
||||
this.subject = subject;
|
||||
this.comparison = comparison;
|
||||
}
|
||||
|
|
|
@ -21,15 +21,14 @@ import java.util.Map;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
|
||||
public class GreaterThanOrEqualEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator comparison;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> comparison;
|
||||
|
||||
public GreaterThanOrEqualEvaluator(final NumberEvaluator subject, final NumberEvaluator comparison) {
|
||||
public GreaterThanOrEqualEvaluator(final Evaluator<Long> subject, final Evaluator<Long> comparison) {
|
||||
this.subject = subject;
|
||||
this.comparison = comparison;
|
||||
}
|
||||
|
|
|
@ -22,14 +22,13 @@ import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public class IndexOfEvaluator extends NumberEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator indexEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> indexEvaluator;
|
||||
|
||||
public IndexOfEvaluator(final StringEvaluator subject, final StringEvaluator indexEvaluator) {
|
||||
public IndexOfEvaluator(final Evaluator<String> subject, final Evaluator<String> indexEvaluator) {
|
||||
this.subject = subject;
|
||||
this.indexEvaluator = indexEvaluator;
|
||||
}
|
||||
|
|
|
@ -22,14 +22,13 @@ import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public class LastIndexOfEvaluator extends NumberEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator indexEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> indexEvaluator;
|
||||
|
||||
public LastIndexOfEvaluator(final StringEvaluator subject, final StringEvaluator indexEvaluator) {
|
||||
public LastIndexOfEvaluator(final Evaluator<String> subject, final Evaluator<String> indexEvaluator) {
|
||||
this.subject = subject;
|
||||
this.indexEvaluator = indexEvaluator;
|
||||
}
|
||||
|
|
|
@ -22,20 +22,19 @@ import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public class LengthEvaluator extends NumberEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final Evaluator<String> subject;
|
||||
|
||||
public LengthEvaluator(final StringEvaluator subject) {
|
||||
public LengthEvaluator(final Evaluator<String> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
@Override
|
||||
public QueryResult<Long> evaluate(final Map<String, String> attributes) {
|
||||
final String subjectValue = subject.evaluate(attributes).getValue();
|
||||
return new NumberQueryResult((long) ((subjectValue == null) ? 0 : subjectValue.length()));
|
||||
return new NumberQueryResult((long) (subjectValue == null ? 0 : subjectValue.length()));
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -21,15 +21,14 @@ import java.util.Map;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
|
||||
public class LessThanEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator comparison;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> comparison;
|
||||
|
||||
public LessThanEvaluator(final NumberEvaluator subject, final NumberEvaluator comparison) {
|
||||
public LessThanEvaluator(final Evaluator<Long> subject, final Evaluator<Long> comparison) {
|
||||
this.subject = subject;
|
||||
this.comparison = comparison;
|
||||
}
|
||||
|
|
|
@ -21,15 +21,14 @@ import java.util.Map;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
|
||||
public class LessThanOrEqualEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator comparison;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> comparison;
|
||||
|
||||
public LessThanOrEqualEvaluator(final NumberEvaluator subject, final NumberEvaluator comparison) {
|
||||
public LessThanOrEqualEvaluator(final Evaluator<Long> subject, final Evaluator<Long> comparison) {
|
||||
this.subject = subject;
|
||||
this.comparison = comparison;
|
||||
}
|
||||
|
|
|
@ -23,17 +23,16 @@ import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.literals.StringLiteralEvaluator;
|
||||
|
||||
public class MatchesEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator search;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> search;
|
||||
|
||||
private final Pattern compiledPattern;
|
||||
|
||||
public MatchesEvaluator(final StringEvaluator subject, final StringEvaluator search) {
|
||||
public MatchesEvaluator(final Evaluator<String> subject, final Evaluator<String> search) {
|
||||
this.subject = subject;
|
||||
this.search = search;
|
||||
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class MinusEvaluator extends NumberEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator minusValue;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> minusValue;
|
||||
|
||||
public MinusEvaluator(final NumberEvaluator subject, final NumberEvaluator minusValue) {
|
||||
public MinusEvaluator(final Evaluator<Long> subject, final Evaluator<Long> minusValue) {
|
||||
this.subject = subject;
|
||||
this.minusValue = minusValue;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class ModEvaluator extends NumberEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator modValue;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> modValue;
|
||||
|
||||
public ModEvaluator(final NumberEvaluator subject, final NumberEvaluator modValue) {
|
||||
public ModEvaluator(final Evaluator<Long> subject, final Evaluator<Long> modValue) {
|
||||
this.subject = subject;
|
||||
this.modValue = modValue;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class MultiplyEvaluator extends NumberEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator multiplyValue;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> multiplyValue;
|
||||
|
||||
public MultiplyEvaluator(final NumberEvaluator subject, final NumberEvaluator multiplyValue) {
|
||||
public MultiplyEvaluator(final Evaluator<Long> subject, final Evaluator<Long> multiplyValue) {
|
||||
this.subject = subject;
|
||||
this.multiplyValue = multiplyValue;
|
||||
}
|
||||
|
|
|
@ -25,9 +25,9 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class NotEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final BooleanEvaluator subjectEvaluator;
|
||||
private final Evaluator<Boolean> subjectEvaluator;
|
||||
|
||||
public NotEvaluator(final BooleanEvaluator subjectEvaluator) {
|
||||
public NotEvaluator(final Evaluator<Boolean> subjectEvaluator) {
|
||||
this.subjectEvaluator = subjectEvaluator;
|
||||
}
|
||||
|
||||
|
|
|
@ -22,14 +22,13 @@ import java.util.Map;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.DateEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.DateQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
|
||||
public class NumberToDateEvaluator extends DateEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final Evaluator<Long> subject;
|
||||
|
||||
public NumberToDateEvaluator(final NumberEvaluator subject) {
|
||||
public NumberToDateEvaluator(final Evaluator<Long> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class OrEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final BooleanEvaluator subjectEvaluator;
|
||||
private final BooleanEvaluator rhsEvaluator;
|
||||
private final Evaluator<Boolean> subjectEvaluator;
|
||||
private final Evaluator<Boolean> rhsEvaluator;
|
||||
|
||||
public OrEvaluator(final BooleanEvaluator subjectEvaluator, final BooleanEvaluator rhsEvaluator) {
|
||||
public OrEvaluator(final Evaluator<Boolean> subjectEvaluator, final Evaluator<Boolean> rhsEvaluator) {
|
||||
this.subjectEvaluator = subjectEvaluator;
|
||||
this.rhsEvaluator = rhsEvaluator;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
|||
|
||||
public class PlusEvaluator extends NumberEvaluator {
|
||||
|
||||
private final NumberEvaluator subject;
|
||||
private final NumberEvaluator plusValue;
|
||||
private final Evaluator<Long> subject;
|
||||
private final Evaluator<Long> plusValue;
|
||||
|
||||
public PlusEvaluator(final NumberEvaluator subject, final NumberEvaluator plusValue) {
|
||||
public PlusEvaluator(final Evaluator<Long> subject, final Evaluator<Long> plusValue) {
|
||||
this.subject = subject;
|
||||
this.plusValue = plusValue;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class PrependEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator prependEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> prependEvaluator;
|
||||
|
||||
public PrependEvaluator(final StringEvaluator subject, final StringEvaluator prepend) {
|
||||
public PrependEvaluator(final Evaluator<String> subject, final Evaluator<String> prepend) {
|
||||
this.subject = subject;
|
||||
this.prependEvaluator = prepend;
|
||||
}
|
||||
|
|
|
@ -25,11 +25,11 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class ReplaceAllEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator search;
|
||||
private final StringEvaluator replacement;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> search;
|
||||
private final Evaluator<String> replacement;
|
||||
|
||||
public ReplaceAllEvaluator(final StringEvaluator subject, final StringEvaluator search, final StringEvaluator replacement) {
|
||||
public ReplaceAllEvaluator(final Evaluator<String> subject, final Evaluator<String> search, final Evaluator<String> replacement) {
|
||||
this.subject = subject;
|
||||
this.search = search;
|
||||
this.replacement = replacement;
|
||||
|
|
|
@ -24,10 +24,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
|||
|
||||
public class ReplaceEmptyEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subjectEvaluator;
|
||||
private final StringEvaluator replacementEvaluator;
|
||||
private final Evaluator<String> subjectEvaluator;
|
||||
private final Evaluator<String> replacementEvaluator;
|
||||
|
||||
public ReplaceEmptyEvaluator(final StringEvaluator subjectEvaluator, final StringEvaluator replacementEvaluator) {
|
||||
public ReplaceEmptyEvaluator(final Evaluator<String> subjectEvaluator, final Evaluator<String> replacementEvaluator) {
|
||||
this.subjectEvaluator = subjectEvaluator;
|
||||
this.replacementEvaluator = replacementEvaluator;
|
||||
}
|
||||
|
|
|
@ -25,11 +25,11 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class ReplaceEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator search;
|
||||
private final StringEvaluator replacement;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> search;
|
||||
private final Evaluator<String> replacement;
|
||||
|
||||
public ReplaceEvaluator(final StringEvaluator subject, final StringEvaluator search, final StringEvaluator replacement) {
|
||||
public ReplaceEvaluator(final Evaluator<String> subject, final Evaluator<String> search, final Evaluator<String> replacement) {
|
||||
this.subject = subject;
|
||||
this.search = search;
|
||||
this.replacement = replacement;
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class ReplaceNullEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator resultEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> resultEvaluator;
|
||||
|
||||
public ReplaceNullEvaluator(final StringEvaluator subject, final StringEvaluator resultEvaluator) {
|
||||
public ReplaceNullEvaluator(final Evaluator<String> subject, final Evaluator<String> resultEvaluator) {
|
||||
this.subject = subject;
|
||||
this.resultEvaluator = resultEvaluator;
|
||||
}
|
||||
|
|
|
@ -22,14 +22,13 @@ import org.apache.nifi.attribute.expression.language.evaluation.BooleanEvaluator
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public class StartsWithEvaluator extends BooleanEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator search;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> search;
|
||||
|
||||
public StartsWithEvaluator(final StringEvaluator subject, final StringEvaluator search) {
|
||||
public StartsWithEvaluator(final Evaluator<String> subject, final Evaluator<String> search) {
|
||||
this.subject = subject;
|
||||
this.search = search;
|
||||
}
|
||||
|
|
|
@ -26,15 +26,14 @@ import org.apache.nifi.attribute.expression.language.evaluation.DateEvaluator;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.DateQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.exception.IllegalAttributeException;
|
||||
|
||||
public class StringToDateEvaluator extends DateEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator format;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> format;
|
||||
|
||||
public StringToDateEvaluator(final StringEvaluator subject, final StringEvaluator format) {
|
||||
public StringToDateEvaluator(final Evaluator<String> subject, final Evaluator<String> format) {
|
||||
this.subject = subject;
|
||||
this.format = format;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class SubstringAfterEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator afterEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> afterEvaluator;
|
||||
|
||||
public SubstringAfterEvaluator(final StringEvaluator subject, final StringEvaluator afterEvaluator) {
|
||||
public SubstringAfterEvaluator(final Evaluator<String> subject, final Evaluator<String> afterEvaluator) {
|
||||
this.subject = subject;
|
||||
this.afterEvaluator = afterEvaluator;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class SubstringAfterLastEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator afterEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> afterEvaluator;
|
||||
|
||||
public SubstringAfterLastEvaluator(final StringEvaluator subject, final StringEvaluator afterEvaluator) {
|
||||
public SubstringAfterLastEvaluator(final Evaluator<String> subject, final Evaluator<String> afterEvaluator) {
|
||||
this.subject = subject;
|
||||
this.afterEvaluator = afterEvaluator;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class SubstringBeforeEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator beforeEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> beforeEvaluator;
|
||||
|
||||
public SubstringBeforeEvaluator(final StringEvaluator subject, final StringEvaluator beforeEvaluator) {
|
||||
public SubstringBeforeEvaluator(final Evaluator<String> subject, final Evaluator<String> beforeEvaluator) {
|
||||
this.subject = subject;
|
||||
this.beforeEvaluator = beforeEvaluator;
|
||||
}
|
||||
|
|
|
@ -25,10 +25,10 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class SubstringBeforeLastEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final StringEvaluator beforeEvaluator;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<String> beforeEvaluator;
|
||||
|
||||
public SubstringBeforeLastEvaluator(final StringEvaluator subject, final StringEvaluator beforeEvaluator) {
|
||||
public SubstringBeforeLastEvaluator(final Evaluator<String> subject, final Evaluator<String> beforeEvaluator) {
|
||||
this.subject = subject;
|
||||
this.beforeEvaluator = beforeEvaluator;
|
||||
}
|
||||
|
|
|
@ -19,24 +19,23 @@ package org.apache.nifi.attribute.expression.language.evaluation.functions;
|
|||
import java.util.Map;
|
||||
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResult;
|
||||
|
||||
public class SubstringEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final NumberEvaluator startIndex;
|
||||
private final NumberEvaluator endIndex;
|
||||
private final Evaluator<String> subject;
|
||||
private final Evaluator<Long> startIndex;
|
||||
private final Evaluator<Long> endIndex;
|
||||
|
||||
public SubstringEvaluator(final StringEvaluator subject, final NumberEvaluator startIndex, final NumberEvaluator endIndex) {
|
||||
public SubstringEvaluator(final Evaluator<String> subject, final Evaluator<Long> startIndex, final Evaluator<Long> endIndex) {
|
||||
this.subject = subject;
|
||||
this.startIndex = startIndex;
|
||||
this.endIndex = endIndex;
|
||||
}
|
||||
|
||||
public SubstringEvaluator(final StringEvaluator subject, final NumberEvaluator startIndex) {
|
||||
public SubstringEvaluator(final Evaluator<String> subject, final Evaluator<Long> startIndex) {
|
||||
this.subject = subject;
|
||||
this.startIndex = startIndex;
|
||||
this.endIndex = null;
|
||||
|
|
|
@ -25,16 +25,16 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class ToLowerEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final Evaluator<String> subject;
|
||||
|
||||
public ToLowerEvaluator(final StringEvaluator subject) {
|
||||
public ToLowerEvaluator(final Evaluator<String> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
@Override
|
||||
public QueryResult<String> evaluate(final Map<String, String> attributes) {
|
||||
final String subjectValue = subject.evaluate(attributes).getValue();
|
||||
return new StringQueryResult((subjectValue == null) ? null : subjectValue.toLowerCase());
|
||||
return new StringQueryResult(subjectValue == null ? null : subjectValue.toLowerCase());
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -22,20 +22,19 @@ import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberQueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public class ToNumberEvaluator extends NumberEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final Evaluator<String> subject;
|
||||
|
||||
public ToNumberEvaluator(final StringEvaluator subject) {
|
||||
public ToNumberEvaluator(final Evaluator<String> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
@Override
|
||||
public QueryResult<Long> evaluate(final Map<String, String> attributes) {
|
||||
final String subjectValue = subject.evaluate(attributes).getValue();
|
||||
return new NumberQueryResult((subjectValue == null || subjectValue.trim().isEmpty()) ? null : Long.valueOf(subjectValue));
|
||||
return new NumberQueryResult(subjectValue == null || subjectValue.trim().isEmpty() ? null : Long.valueOf(subjectValue));
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -20,22 +20,21 @@ import java.util.Arrays;
|
|||
import java.util.Map;
|
||||
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.NumberEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResult;
|
||||
|
||||
public class ToRadixEvaluator extends StringEvaluator {
|
||||
|
||||
private final NumberEvaluator numberEvaluator;
|
||||
private final NumberEvaluator radixEvaluator;
|
||||
private final NumberEvaluator minimumWidthEvaluator;
|
||||
private final Evaluator<Long> numberEvaluator;
|
||||
private final Evaluator<Long> radixEvaluator;
|
||||
private final Evaluator<Long> minimumWidthEvaluator;
|
||||
|
||||
public ToRadixEvaluator(final NumberEvaluator subject, final NumberEvaluator radixEvaluator) {
|
||||
public ToRadixEvaluator(final Evaluator<Long> subject, final Evaluator<Long> radixEvaluator) {
|
||||
this(subject, radixEvaluator, null);
|
||||
}
|
||||
|
||||
public ToRadixEvaluator(final NumberEvaluator subject, final NumberEvaluator radixEvaluator, final NumberEvaluator minimumWidthEvaluator) {
|
||||
public ToRadixEvaluator(final Evaluator<Long> subject, final Evaluator<Long> radixEvaluator, final Evaluator<Long> minimumWidthEvaluator) {
|
||||
this.numberEvaluator = subject;
|
||||
this.radixEvaluator = radixEvaluator;
|
||||
this.minimumWidthEvaluator = minimumWidthEvaluator;
|
||||
|
|
|
@ -25,16 +25,16 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class ToUpperEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final Evaluator<String> subject;
|
||||
|
||||
public ToUpperEvaluator(final StringEvaluator subject) {
|
||||
public ToUpperEvaluator(final Evaluator<String> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
@Override
|
||||
public QueryResult<String> evaluate(final Map<String, String> attributes) {
|
||||
final String subjectValue = subject.evaluate(attributes).getValue();
|
||||
return new StringQueryResult((subjectValue == null) ? null : subjectValue.toUpperCase());
|
||||
return new StringQueryResult(subjectValue == null ? null : subjectValue.toUpperCase());
|
||||
}
|
||||
|
||||
@Override
|
||||
|
|
|
@ -25,9 +25,9 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class TrimEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final Evaluator<String> subject;
|
||||
|
||||
public TrimEvaluator(final StringEvaluator subject) {
|
||||
public TrimEvaluator(final Evaluator<String> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
|
|
|
@ -27,9 +27,9 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class UrlDecodeEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final Evaluator<String> subject;
|
||||
|
||||
public UrlDecodeEvaluator(final StringEvaluator subject) {
|
||||
public UrlDecodeEvaluator(final Evaluator<String> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
|
@ -42,7 +42,7 @@ public class UrlDecodeEvaluator extends StringEvaluator {
|
|||
|
||||
try {
|
||||
return new StringQueryResult(URLDecoder.decode(subjectValue, "UTF-8"));
|
||||
} catch (UnsupportedEncodingException e) {
|
||||
} catch (final UnsupportedEncodingException e) {
|
||||
return null; // won't happen. It's UTF-8
|
||||
}
|
||||
}
|
||||
|
|
|
@ -27,9 +27,9 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class UrlEncodeEvaluator extends StringEvaluator {
|
||||
|
||||
private final StringEvaluator subject;
|
||||
private final Evaluator<String> subject;
|
||||
|
||||
public UrlEncodeEvaluator(final StringEvaluator subject) {
|
||||
public UrlEncodeEvaluator(final Evaluator<String> subject) {
|
||||
this.subject = subject;
|
||||
}
|
||||
|
||||
|
@ -42,7 +42,7 @@ public class UrlEncodeEvaluator extends StringEvaluator {
|
|||
|
||||
try {
|
||||
return new StringQueryResult(URLEncoder.encode(subjectValue, "UTF-8"));
|
||||
} catch (UnsupportedEncodingException e) {
|
||||
} catch (final UnsupportedEncodingException e) {
|
||||
return null; // won't happen. It's UTF-8
|
||||
}
|
||||
}
|
||||
|
|
|
@ -0,0 +1,43 @@
|
|||
/*
|
||||
* 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.attribute.expression.language.evaluation.literals;
|
||||
|
||||
import java.util.Map;
|
||||
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResult;
|
||||
|
||||
public class ToLiteralEvaluator extends StringEvaluator {
|
||||
private final Evaluator<?> argEvaluator;
|
||||
|
||||
public ToLiteralEvaluator(final Evaluator<?> argEvaluator) {
|
||||
this.argEvaluator = argEvaluator;
|
||||
}
|
||||
|
||||
@Override
|
||||
public QueryResult<String> evaluate(final Map<String, String> attributes) {
|
||||
final Object result = argEvaluator.evaluate(attributes);
|
||||
return new StringQueryResult(result == null ? null : result.toString());
|
||||
}
|
||||
|
||||
@Override
|
||||
public Evaluator<?> getSubjectEvaluator() {
|
||||
return null;
|
||||
}
|
||||
}
|
|
@ -25,13 +25,13 @@ import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResul
|
|||
|
||||
public class JoinEvaluator extends StringEvaluator implements ReduceEvaluator<String> {
|
||||
|
||||
private final StringEvaluator subjectEvaluator;
|
||||
private final StringEvaluator delimiterEvaluator;
|
||||
private final Evaluator<String> subjectEvaluator;
|
||||
private final Evaluator<String> delimiterEvaluator;
|
||||
|
||||
private final StringBuilder sb = new StringBuilder();
|
||||
private int evalCount = 0;
|
||||
|
||||
public JoinEvaluator(final StringEvaluator subject, final StringEvaluator delimiter) {
|
||||
public JoinEvaluator(final Evaluator<String> subject, final Evaluator<String> delimiter) {
|
||||
this.subjectEvaluator = subject;
|
||||
this.delimiterEvaluator = delimiter;
|
||||
}
|
||||
|
|
|
@ -23,7 +23,7 @@ import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResu
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
|
||||
public class AllAttributesEvaluator extends BooleanEvaluator {
|
||||
public class AllAttributesEvaluator extends BooleanEvaluator implements IteratingEvaluator<Boolean> {
|
||||
|
||||
private final BooleanEvaluator booleanEvaluator;
|
||||
private final MultiAttributeEvaluator multiAttributeEvaluator;
|
||||
|
@ -65,4 +65,9 @@ public class AllAttributesEvaluator extends BooleanEvaluator {
|
|||
public Evaluator<?> getSubjectEvaluator() {
|
||||
return null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Evaluator<?> getLogicEvaluator() {
|
||||
return booleanEvaluator;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -23,7 +23,7 @@ import org.apache.nifi.attribute.expression.language.evaluation.BooleanQueryResu
|
|||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
|
||||
public class AnyAttributeEvaluator extends BooleanEvaluator {
|
||||
public class AnyAttributeEvaluator extends BooleanEvaluator implements IteratingEvaluator<Boolean> {
|
||||
|
||||
private final BooleanEvaluator booleanEvaluator;
|
||||
private final MultiAttributeEvaluator multiAttributeEvaluator;
|
||||
|
@ -65,4 +65,9 @@ public class AnyAttributeEvaluator extends BooleanEvaluator {
|
|||
public Evaluator<?> getSubjectEvaluator() {
|
||||
return null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Evaluator<Boolean> getLogicEvaluator() {
|
||||
return booleanEvaluator;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -20,19 +20,18 @@ import java.util.Map;
|
|||
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.QueryResult;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringQueryResult;
|
||||
|
||||
public class DelineatedAttributeEvaluator extends MultiAttributeEvaluator {
|
||||
|
||||
private final StringEvaluator subjectEvaluator;
|
||||
private final StringEvaluator delimiterEvaluator;
|
||||
private final Evaluator<String> subjectEvaluator;
|
||||
private final Evaluator<String> delimiterEvaluator;
|
||||
private final int evaluationType;
|
||||
private String[] delineatedValues;
|
||||
private int evaluationCount = 0;
|
||||
private int evaluationsLeft = 1;
|
||||
|
||||
public DelineatedAttributeEvaluator(final StringEvaluator subjectEvaluator, final StringEvaluator delimiterEvaluator, final int evaluationType) {
|
||||
public DelineatedAttributeEvaluator(final Evaluator<String> subjectEvaluator, final Evaluator<String> delimiterEvaluator, final int evaluationType) {
|
||||
this.subjectEvaluator = subjectEvaluator;
|
||||
this.delimiterEvaluator = delimiterEvaluator;
|
||||
this.evaluationType = evaluationType;
|
||||
|
@ -66,6 +65,11 @@ public class DelineatedAttributeEvaluator extends MultiAttributeEvaluator {
|
|||
return new StringQueryResult(delineatedValues[evaluationCount++]);
|
||||
}
|
||||
|
||||
@Override
|
||||
public Evaluator<?> getLogicEvaluator() {
|
||||
return subjectEvaluator;
|
||||
}
|
||||
|
||||
@Override
|
||||
public int getEvaluationsRemaining() {
|
||||
return evaluationsLeft;
|
||||
|
|
|
@ -16,6 +16,18 @@
|
|||
*/
|
||||
package org.apache.nifi.attribute.expression.language.evaluation.selection;
|
||||
|
||||
public class AnyMatchingAttributeEvaluator {
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.Evaluator;
|
||||
|
||||
/**
|
||||
* Interface for an Evaluator that should be evaluated multiple times
|
||||
*
|
||||
* @param <T> return type of evaluator
|
||||
*/
|
||||
public interface IteratingEvaluator<T> extends Evaluator<T> {
|
||||
|
||||
/**
|
||||
* @return the evaluator that evaluates some sort of logic against its subject
|
||||
*/
|
||||
Evaluator<?> getLogicEvaluator();
|
||||
|
||||
}
|
|
@ -24,9 +24,9 @@ import org.apache.nifi.attribute.expression.language.evaluation.reduce.ReduceEva
|
|||
import org.apache.nifi.expression.AttributeExpression.ResultType;
|
||||
|
||||
public class MappingEvaluator<T> implements Evaluator<T> {
|
||||
|
||||
private final ReduceEvaluator<T> mappingEvaluator;
|
||||
private final MultiAttributeEvaluator multiAttributeEvaluator;
|
||||
private String token;
|
||||
|
||||
public MappingEvaluator(final ReduceEvaluator<T> mappingEvaluator, final MultiAttributeEvaluator multiAttributeEval) {
|
||||
this.mappingEvaluator = mappingEvaluator;
|
||||
|
@ -59,4 +59,13 @@ public class MappingEvaluator<T> implements Evaluator<T> {
|
|||
return null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String getToken() {
|
||||
return token;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void setToken(final String token) {
|
||||
this.token = token;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -18,7 +18,6 @@ package org.apache.nifi.attribute.expression.language.evaluation.selection;
|
|||
|
||||
import org.apache.nifi.attribute.expression.language.evaluation.StringEvaluator;
|
||||
|
||||
public abstract class MultiAttributeEvaluator extends StringEvaluator {
|
||||
|
||||
public abstract class MultiAttributeEvaluator extends StringEvaluator implements IteratingEvaluator<String> {
|
||||
public abstract int getEvaluationType();
|
||||
}
|
||||
|
|
|
@ -79,4 +79,9 @@ public class MultiMatchAttributeEvaluator extends MultiAttributeEvaluator {
|
|||
public int getEvaluationType() {
|
||||
return evaluationType;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Evaluator<?> getLogicEvaluator() {
|
||||
return this;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -61,4 +61,9 @@ public class MultiNamedAttributeEvaluator extends MultiAttributeEvaluator {
|
|||
public int getEvaluationType() {
|
||||
return evaluationType;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Evaluator<?> getLogicEvaluator() {
|
||||
return this;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -17,10 +17,12 @@
|
|||
package org.apache.nifi.attribute.expression.language;
|
||||
|
||||
import static org.junit.Assert.assertEquals;
|
||||
import static org.junit.Assert.assertFalse;
|
||||
import static org.junit.Assert.assertTrue;
|
||||
|
||||
import java.text.SimpleDateFormat;
|
||||
import java.util.Calendar;
|
||||
import java.util.Collections;
|
||||
import java.util.Date;
|
||||
import java.util.HashMap;
|
||||
import java.util.HashSet;
|
||||
|
@ -52,6 +54,8 @@ public class TestQuery {
|
|||
assertValid("${now():format('yyyy/MM/dd')}");
|
||||
assertInvalid("${attr:times(3)}");
|
||||
assertValid("${attr:toNumber():multiply(3)}");
|
||||
assertValid("${hostname()}");
|
||||
assertValid("${literal(3)}");
|
||||
// left here because it's convenient for looking at the output
|
||||
//System.out.println(Query.compile("").evaluate(null));
|
||||
}
|
||||
|
@ -296,7 +300,7 @@ public class TestQuery {
|
|||
}
|
||||
|
||||
private String evaluateQueryForEscape(final String queryString, final Map<String, String> attributes) {
|
||||
FlowFile mockFlowFile = Mockito.mock(FlowFile.class);
|
||||
final FlowFile mockFlowFile = Mockito.mock(FlowFile.class);
|
||||
Mockito.when(mockFlowFile.getAttributes()).thenReturn(attributes);
|
||||
Mockito.when(mockFlowFile.getId()).thenReturn(1L);
|
||||
Mockito.when(mockFlowFile.getEntryDate()).thenReturn(System.currentTimeMillis());
|
||||
|
@ -429,7 +433,7 @@ public class TestQuery {
|
|||
|
||||
@Test
|
||||
public void testExtractExpressionTypes() {
|
||||
List<ResultType> types = Query.extractResultTypes("${hello:equals( ${goodbye} )} or just hi, ${bob}, are you ${bob.age:toNumber()} yet? $$$${bob}");
|
||||
final List<ResultType> types = Query.extractResultTypes("${hello:equals( ${goodbye} )} or just hi, ${bob}, are you ${bob.age:toNumber()} yet? $$$${bob}");
|
||||
assertEquals(3, types.size());
|
||||
assertEquals(ResultType.BOOLEAN, types.get(0));
|
||||
assertEquals(ResultType.STRING, types.get(1));
|
||||
|
@ -493,7 +497,7 @@ public class TestQuery {
|
|||
final Map<String, String> attributes = new HashMap<>();
|
||||
attributes.put("xx", "say 'hi'");
|
||||
|
||||
String query = "${xx:replace( \"'hi'\", '\\\"hello\\\"' )}";
|
||||
final String query = "${xx:replace( \"'hi'\", '\\\"hello\\\"' )}";
|
||||
System.out.println(query);
|
||||
verifyEquals(query, attributes, "say \"hello\"");
|
||||
}
|
||||
|
@ -1112,12 +1116,52 @@ public class TestQuery {
|
|||
assertEquals("{ xyz }", Query.evaluateExpressions(query, attributes));
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testLiteralFunction() {
|
||||
final Map<String, String> attrs = Collections.<String, String> emptyMap();
|
||||
verifyEquals("${literal(2):gt(1)}", attrs, true);
|
||||
verifyEquals("${literal('hello'):substring(0, 1):equals('h')}", attrs, true);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testFunctionAfterReduce() {
|
||||
// Cannot call gt(2) after count() because count() is a 'reducing function'
|
||||
// and must be the last function in an expression.
|
||||
assertFalse(Query.isValidExpression("${allMatchingAttributes('a.*'):contains('2'):count():gt(2)}"));
|
||||
|
||||
final Map<String, String> attributes = new HashMap<>();
|
||||
attributes.put("a.1", "245");
|
||||
attributes.put("a.2", "123");
|
||||
attributes.put("a.3", "732");
|
||||
attributes.put("a.4", "343");
|
||||
attributes.put("a.5", "553");
|
||||
|
||||
final String endsWithCount = "${allMatchingAttributes('a.*'):contains('2'):count()}";
|
||||
assertTrue(Query.isValidExpression(endsWithCount));
|
||||
verifyEquals(endsWithCount, attributes, 3L);
|
||||
|
||||
// in order to check if value is greater than 2, need to first evaluate the
|
||||
// 'aggregate' and 'reducing' functions as an inner expression. Then we can
|
||||
// use the literal() function to make the result of the inner expression the subject
|
||||
// of the function gt()
|
||||
final String usingLiteral = "${literal(" + endsWithCount + "):gt(2)}";
|
||||
assertTrue(Query.isValidExpression(usingLiteral));
|
||||
verifyEquals(usingLiteral, attributes, true);
|
||||
|
||||
attributes.clear();
|
||||
attributes.put("a1", "123");
|
||||
attributes.put("a2", "321");
|
||||
verifyEquals("${allMatchingAttributes('a.*'):contains('2')}", attributes, true);
|
||||
verifyEquals("${allMatchingAttributes('a.*'):contains('2'):toUpper():equals('TRUE')}", attributes, true);
|
||||
verifyEquals("${allMatchingAttributes('a.*'):contains('2'):equals('true'):and( ${literal(true)} )}", attributes, true);
|
||||
}
|
||||
|
||||
private void verifyEquals(final String expression, final Map<String, String> attributes, final Object expectedResult) {
|
||||
Query.validateExpression(expression, false);
|
||||
assertEquals(String.valueOf(expectedResult), Query.evaluateExpressions(expression, attributes, null));
|
||||
|
||||
Query query = Query.compile(expression);
|
||||
QueryResult<?> result = query.evaluate(attributes);
|
||||
final Query query = Query.compile(expression);
|
||||
final QueryResult<?> result = query.evaluate(attributes);
|
||||
|
||||
if (expectedResult instanceof Number) {
|
||||
assertEquals(ResultType.NUMBER, result.getResultType());
|
||||
|
|
|
@ -233,8 +233,7 @@ public class NiFiProperties extends Properties {
|
|||
* obtained.
|
||||
*
|
||||
* @return the NiFiProperties object to use
|
||||
* @throws RuntimeException
|
||||
* if unable to load properties file
|
||||
* @throws RuntimeException if unable to load properties file
|
||||
*/
|
||||
public static synchronized NiFiProperties getInstance() {
|
||||
if (null == instance) {
|
||||
|
@ -794,7 +793,7 @@ public class NiFiProperties extends Properties {
|
|||
final String scheme = (rawScheme == null) ? "http" : rawScheme;
|
||||
|
||||
final String host;
|
||||
final int port;
|
||||
final Integer port;
|
||||
if ("http".equalsIgnoreCase(scheme)) {
|
||||
// get host
|
||||
if (StringUtils.isBlank(getProperty(WEB_HTTP_HOST))) {
|
||||
|
@ -804,6 +803,10 @@ public class NiFiProperties extends Properties {
|
|||
}
|
||||
// get port
|
||||
port = getPort();
|
||||
|
||||
if (port == null) {
|
||||
throw new RuntimeException(String.format("The %s must be specified if running in a cluster with %s set to false.", WEB_HTTP_PORT, CLUSTER_PROTOCOL_IS_SECURE));
|
||||
}
|
||||
} else {
|
||||
// get host
|
||||
if (StringUtils.isBlank(getProperty(WEB_HTTPS_HOST))) {
|
||||
|
@ -813,6 +816,10 @@ public class NiFiProperties extends Properties {
|
|||
}
|
||||
// get port
|
||||
port = getSslPort();
|
||||
|
||||
if (port == null) {
|
||||
throw new RuntimeException(String.format("The %s must be specified if running in a cluster with %s set to true.", WEB_HTTPS_PORT, CLUSTER_PROTOCOL_IS_SECURE));
|
||||
}
|
||||
}
|
||||
|
||||
return InetSocketAddress.createUnresolved(host, port);
|
||||
|
@ -824,8 +831,7 @@ public class NiFiProperties extends Properties {
|
|||
* configured. No directories will be created as a result of this operation.
|
||||
*
|
||||
* @return database repository path
|
||||
* @throws InvalidPathException
|
||||
* If the configured path is invalid
|
||||
* @throws InvalidPathException If the configured path is invalid
|
||||
*/
|
||||
public Path getDatabaseRepositoryPath() {
|
||||
return Paths.get(getProperty(REPOSITORY_DATABASE_DIRECTORY));
|
||||
|
@ -836,8 +842,7 @@ public class NiFiProperties extends Properties {
|
|||
* configured. No directories will be created as a result of this operation.
|
||||
*
|
||||
* @return database repository path
|
||||
* @throws InvalidPathException
|
||||
* If the configured path is invalid
|
||||
* @throws InvalidPathException If the configured path is invalid
|
||||
*/
|
||||
public Path getFlowFileRepositoryPath() {
|
||||
return Paths.get(getProperty(FLOWFILE_REPOSITORY_DIRECTORY));
|
||||
|
@ -850,8 +855,7 @@ public class NiFiProperties extends Properties {
|
|||
* operation.
|
||||
*
|
||||
* @return file repositories paths
|
||||
* @throws InvalidPathException
|
||||
* If any of the configured paths are invalid
|
||||
* @throws InvalidPathException If any of the configured paths are invalid
|
||||
*/
|
||||
public Map<String, Path> getContentRepositoryPaths() {
|
||||
final Map<String, Path> contentRepositoryPaths = new HashMap<>();
|
||||
|
|
|
@ -210,7 +210,7 @@ Here is an example entry using the name John Smith:
|
|||
|
||||
----
|
||||
<users>
|
||||
<user dn="[cn=John Smith,ou=people,dc=example,dc=com]">
|
||||
<user dn="cn=John Smith,ou=people,dc=example,dc=com">
|
||||
<role name="ROLE_ADMIN"/>
|
||||
</user>
|
||||
</users>
|
||||
|
|
|
@ -1317,9 +1317,9 @@ Expressions will provide the following results:
|
|||
and Midnight GMT on January 1, 2014. The Expression `${time:toDate("yyyy/MM/dd HH:mm:ss.SSS'Z'")}` will result in the
|
||||
number of milliseconds between Midnight GMT on January 1, 1970 and 15:36:03.264 GMT on December 31, 2014.
|
||||
|
||||
Often, this function is used in conjunction with the <<format>> function to change the format of a date/time. For example,
|
||||
if the attribute "date" has the value "12-24-2014" and we want to change the format to "2014/12/24", we can do so by
|
||||
chaining together the two functions: `${date:toDate('MM-dd-yyyy'):format('yyyy/MM/dd')}`.
|
||||
Often, this function is used in conjunction with the <<format>> function to change the format of a date/time. For example,
|
||||
if the attribute "date" has the value "12-24-2014" and we want to change the format to "2014/12/24", we can do so by
|
||||
chaining together the two functions: `${date:toDate('MM-dd-yyyy'):format('yyyy/MM/dd')}`.
|
||||
|
||||
|
||||
|
||||
|
@ -1404,7 +1404,7 @@ an error when validating the function.
|
|||
|
||||
*Return Type*: [.returnType]#String#
|
||||
|
||||
*Examples*: The IP address of the machine can be obtained by using the Expresison `${ip()}`.
|
||||
*Examples*: The IP address of the machine can be obtained by using the Expression `${ip()}`.
|
||||
|
||||
|
||||
|
||||
|
@ -1471,7 +1471,25 @@ an error when validating the function.
|
|||
|
||||
|
||||
|
||||
[.function]
|
||||
=== literal
|
||||
|
||||
*Description*: [.description]#Returns its argument as a literal String value. This is useful in order to treat a string or a number
|
||||
at the beginning of an Expression as an actual value, rather than treating it as an attribute name. Additionally, it
|
||||
can be used when the argument is an embedded Expression that we would then like to evaluate additional functions against.#
|
||||
|
||||
*Subject Type*: [.subjectless]#No Subject#
|
||||
|
||||
*Arguments*:
|
||||
|
||||
- [.argName]#_value_# : [.argDesc]#The value to be treated as a literal string, number, or boolean value.#
|
||||
|
||||
*Return Type*: [.returnType]#String#
|
||||
|
||||
*Examples*: `${literal(2):gt(1)}` returns `true`
|
||||
|
||||
`${literal( ${allMatchingAttributes('a.*'):count()} ):gt(3)}` returns true if there are more than 3 attributes whose
|
||||
names begin with the letter `a`.
|
||||
|
||||
|
||||
[[multi]]
|
||||
|
|
|
@ -0,0 +1,36 @@
|
|||
<?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.
|
||||
-->
|
||||
<project xmlns="http://maven.apache.org/POM/4.0.0" xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance" xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd">
|
||||
<modelVersion>4.0.0</modelVersion>
|
||||
|
||||
<parent>
|
||||
<groupId>org.apache.nifi</groupId>
|
||||
<artifactId>nifi-example-bundle</artifactId>
|
||||
<version>0.1.1-incubating-SNAPSHOT</version>
|
||||
</parent>
|
||||
|
||||
<artifactId>nifi-nifi-example-nar</artifactId>
|
||||
<packaging>nar</packaging>
|
||||
|
||||
<dependencies>
|
||||
<dependency>
|
||||
<groupId>org.apache.nifi</groupId>
|
||||
<artifactId>nifi-nifi-example-processors</artifactId>
|
||||
<version>${project.version}</version>
|
||||
</dependency>
|
||||
</dependencies>
|
||||
|
||||
</project>
|
|
@ -0,0 +1,70 @@
|
|||
<?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.
|
||||
-->
|
||||
<project xmlns="http://maven.apache.org/POM/4.0.0" xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance" xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd">
|
||||
<modelVersion>4.0.0</modelVersion>
|
||||
|
||||
<parent>
|
||||
<groupId>org.apache.nifi</groupId>
|
||||
<artifactId>nifi-example-bundle</artifactId>
|
||||
<version>0.1.1-incubating-SNAPSHOT</version>
|
||||
</parent>
|
||||
|
||||
<artifactId>nifi-nifi-example-processors</artifactId>
|
||||
<packaging>jar</packaging>
|
||||
|
||||
<dependencies>
|
||||
<dependency>
|
||||
<groupId>org.apache.nifi</groupId>
|
||||
<artifactId>nifi-api</artifactId>
|
||||
</dependency>
|
||||
<dependency>
|
||||
<groupId>org.apache.nifi</groupId>
|
||||
<artifactId>nifi-processor-utils</artifactId>
|
||||
</dependency>
|
||||
<dependency>
|
||||
<groupId>org.apache.nifi</groupId>
|
||||
<artifactId>nifi-mock</artifactId>
|
||||
<scope>test</scope>
|
||||
</dependency>
|
||||
<dependency>
|
||||
<groupId>org.slf4j</groupId>
|
||||
<artifactId>slf4j-simple</artifactId>
|
||||
<scope>test</scope>
|
||||
</dependency>
|
||||
<dependency>
|
||||
<groupId>commons-io</groupId>
|
||||
<artifactId>commons-io</artifactId>
|
||||
</dependency>
|
||||
<dependency>
|
||||
<groupId>junit</groupId>
|
||||
<artifactId>junit</artifactId>
|
||||
<scope>test</scope>
|
||||
</dependency>
|
||||
</dependencies>
|
||||
<build>
|
||||
<plugins>
|
||||
<plugin>
|
||||
<groupId>org.apache.rat</groupId>
|
||||
<artifactId>apache-rat-plugin</artifactId>
|
||||
<configuration>
|
||||
<excludes>
|
||||
<exclude>src/main/resources/file.txt</exclude>
|
||||
</excludes>
|
||||
</configuration>
|
||||
</plugin>
|
||||
</plugins>
|
||||
</build>
|
||||
</project>
|
|
@ -0,0 +1,102 @@
|
|||
/*
|
||||
* 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.processors;
|
||||
|
||||
import java.io.IOException;
|
||||
import java.io.OutputStream;
|
||||
import java.util.Collections;
|
||||
import java.util.HashSet;
|
||||
import java.util.Set;
|
||||
|
||||
import org.apache.commons.io.IOUtils;
|
||||
import org.apache.nifi.annotation.documentation.CapabilityDescription;
|
||||
import org.apache.nifi.annotation.documentation.Tags;
|
||||
import org.apache.nifi.annotation.lifecycle.OnScheduled;
|
||||
import org.apache.nifi.flowfile.FlowFile;
|
||||
import org.apache.nifi.processor.AbstractProcessor;
|
||||
import org.apache.nifi.processor.ProcessContext;
|
||||
import org.apache.nifi.processor.ProcessSession;
|
||||
import org.apache.nifi.processor.ProcessorInitializationContext;
|
||||
import org.apache.nifi.processor.Relationship;
|
||||
import org.apache.nifi.processor.exception.ProcessException;
|
||||
import org.apache.nifi.processor.io.OutputStreamCallback;
|
||||
|
||||
@Tags({ "example", "resources" })
|
||||
@CapabilityDescription("This example processor loads a resource from the nar and writes it to the FlowFile content")
|
||||
public class WriteResourceToStream extends AbstractProcessor {
|
||||
|
||||
public static final Relationship REL_SUCCESS = new Relationship.Builder()
|
||||
.name("success")
|
||||
.description("files that were successfully processed").build();
|
||||
public static final Relationship REL_FAILURE = new Relationship.Builder()
|
||||
.name("failure")
|
||||
.description("files that were not successfully processed").build();
|
||||
|
||||
private Set<Relationship> relationships;
|
||||
|
||||
private String resourceData;
|
||||
|
||||
@Override
|
||||
protected void init(final ProcessorInitializationContext context) {
|
||||
|
||||
final Set<Relationship> relationships = new HashSet<Relationship>();
|
||||
relationships.add(REL_SUCCESS);
|
||||
relationships.add(REL_FAILURE);
|
||||
this.relationships = Collections.unmodifiableSet(relationships);
|
||||
|
||||
try {
|
||||
this.resourceData = IOUtils.toString(Thread.currentThread()
|
||||
.getContextClassLoader().getResourceAsStream("file.txt"));
|
||||
} catch (IOException e) {
|
||||
throw new RuntimeException("Unable to load resources", e);
|
||||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public Set<Relationship> getRelationships() {
|
||||
return this.relationships;
|
||||
}
|
||||
|
||||
@OnScheduled
|
||||
public void onScheduled(final ProcessContext context) {
|
||||
|
||||
}
|
||||
|
||||
@Override
|
||||
public void onTrigger(final ProcessContext context,
|
||||
final ProcessSession session) throws ProcessException {
|
||||
FlowFile flowFile = session.get();
|
||||
if (flowFile == null) {
|
||||
return;
|
||||
}
|
||||
|
||||
try {
|
||||
flowFile = session.write(flowFile, new OutputStreamCallback() {
|
||||
|
||||
@Override
|
||||
public void process(OutputStream out) throws IOException {
|
||||
IOUtils.write(resourceData, out);
|
||||
|
||||
}
|
||||
});
|
||||
session.transfer(flowFile, REL_SUCCESS);
|
||||
} catch (ProcessException ex) {
|
||||
getLogger().error("Unable to process", ex);
|
||||
session.transfer(flowFile, REL_FAILURE);
|
||||
}
|
||||
}
|
||||
}
|
|
@ -0,0 +1,15 @@
|
|||
# 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.
|
||||
org.apache.nifi.processors.WriteResourceToStream
|
|
@ -0,0 +1 @@
|
|||
this came from a resource
|
|
@ -0,0 +1,47 @@
|
|||
/*
|
||||
* 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.processors;
|
||||
|
||||
import org.junit.Assert;
|
||||
|
||||
import org.apache.nifi.util.TestRunner;
|
||||
import org.apache.nifi.util.TestRunners;
|
||||
import org.junit.Before;
|
||||
import org.junit.Test;
|
||||
|
||||
public class WriteResourceToStreamTest {
|
||||
|
||||
private TestRunner testRunner;
|
||||
|
||||
@Before
|
||||
public void init() {
|
||||
testRunner = TestRunners.newTestRunner(WriteResourceToStream.class);
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testProcessor() {
|
||||
testRunner.enqueue(new byte[] { 1, 2, 3, 4, 5 });
|
||||
testRunner.run();
|
||||
testRunner.assertAllFlowFilesTransferred(WriteResourceToStream.REL_SUCCESS, 1);
|
||||
final byte[] data = testRunner
|
||||
.getFlowFilesForRelationship(WriteResourceToStream.REL_SUCCESS).get(0)
|
||||
.toByteArray();
|
||||
final String stringData = new String(data);
|
||||
Assert.assertEquals("this came from a resource", stringData);
|
||||
}
|
||||
|
||||
}
|
|
@ -0,0 +1,33 @@
|
|||
<?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.
|
||||
-->
|
||||
<project xmlns="http://maven.apache.org/POM/4.0.0" xmlns:xsi="http://www.w3.org/2001/XMLSchema-instance" xsi:schemaLocation="http://maven.apache.org/POM/4.0.0 http://maven.apache.org/xsd/maven-4.0.0.xsd">
|
||||
<modelVersion>4.0.0</modelVersion>
|
||||
|
||||
<parent>
|
||||
<groupId>org.apache.nifi</groupId>
|
||||
<artifactId>nifi-external</artifactId>
|
||||
<version>0.1.1-incubating-SNAPSHOT</version>
|
||||
</parent>
|
||||
|
||||
<artifactId>nifi-example-bundle</artifactId>
|
||||
<packaging>pom</packaging>
|
||||
|
||||
<modules>
|
||||
<module>nifi-nifi-example-processors</module>
|
||||
<module>nifi-nifi-example-nar</module>
|
||||
</modules>
|
||||
|
||||
</project>
|
|
@ -26,5 +26,7 @@
|
|||
<modules>
|
||||
<module>nifi-spark-receiver</module>
|
||||
<module>nifi-storm-spout</module>
|
||||
<module>nifi-example-bundle</module>
|
||||
</modules>
|
||||
</project>
|
||||
|
||||
|
|
|
@ -22,14 +22,21 @@ import java.util.Map;
|
|||
import org.apache.nifi.components.PropertyDescriptor;
|
||||
import org.apache.nifi.components.PropertyValue;
|
||||
import org.apache.nifi.controller.ConfigurationContext;
|
||||
import org.apache.nifi.controller.ControllerService;
|
||||
import org.apache.nifi.controller.ControllerServiceLookup;
|
||||
|
||||
public class MockConfigurationContext implements ConfigurationContext {
|
||||
|
||||
private final Map<PropertyDescriptor, String> properties;
|
||||
private final ControllerServiceLookup serviceLookup;
|
||||
private final ControllerService service;
|
||||
|
||||
public MockConfigurationContext(final Map<PropertyDescriptor, String> properties, final ControllerServiceLookup serviceLookup) {
|
||||
this(null, properties, serviceLookup);
|
||||
}
|
||||
|
||||
public MockConfigurationContext(final ControllerService service, final Map<PropertyDescriptor, String> properties, final ControllerServiceLookup serviceLookup) {
|
||||
this.service = service;
|
||||
this.properties = properties;
|
||||
this.serviceLookup = serviceLookup;
|
||||
}
|
||||
|
@ -38,7 +45,7 @@ public class MockConfigurationContext implements ConfigurationContext {
|
|||
public PropertyValue getProperty(final PropertyDescriptor property) {
|
||||
String value = properties.get(property);
|
||||
if (value == null) {
|
||||
value = property.getDefaultValue();
|
||||
value = getActualDescriptor(property).getDefaultValue();
|
||||
}
|
||||
return new MockPropertyValue(value, serviceLookup);
|
||||
}
|
||||
|
@ -47,4 +54,13 @@ public class MockConfigurationContext implements ConfigurationContext {
|
|||
public Map<PropertyDescriptor, String> getProperties() {
|
||||
return new HashMap<>(this.properties);
|
||||
}
|
||||
|
||||
private PropertyDescriptor getActualDescriptor(final PropertyDescriptor property) {
|
||||
if (service == null) {
|
||||
return property;
|
||||
}
|
||||
|
||||
final PropertyDescriptor resolved = service.getPropertyDescriptor(property.getName());
|
||||
return resolved == null ? property : resolved;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -214,8 +214,8 @@ public class StandardProcessorTestRunner implements TestRunner {
|
|||
|
||||
executorService.shutdown();
|
||||
try {
|
||||
executorService.awaitTermination(runWait, TimeUnit.SECONDS);
|
||||
} catch (InterruptedException e1) {
|
||||
executorService.awaitTermination(runWait, TimeUnit.MILLISECONDS);
|
||||
} catch (final InterruptedException e1) {
|
||||
}
|
||||
|
||||
int finishedCount = 0;
|
||||
|
@ -609,7 +609,7 @@ public class StandardProcessorTestRunner implements TestRunner {
|
|||
}
|
||||
|
||||
try {
|
||||
final ConfigurationContext configContext = new MockConfigurationContext(configuration.getProperties(), context);
|
||||
final ConfigurationContext configContext = new MockConfigurationContext(service, configuration.getProperties(), context);
|
||||
ReflectionUtils.invokeMethodsWithAnnotation(OnEnabled.class, service, configContext);
|
||||
} catch (final InvocationTargetException ite) {
|
||||
ite.getCause().printStackTrace();
|
||||
|
|
|
@ -165,7 +165,7 @@ public interface TestRunner {
|
|||
* @param stopOnFinish whether or not to run the Processor methods that are
|
||||
* annotated with {@link nifi.processor.annotation.OnStopped @OnStopped}
|
||||
* @param initialize true if must initialize
|
||||
* @param runWait indiciates the amount of time in milliseconds that the framework should wait for
|
||||
* @param runWait indicates the amount of time in milliseconds that the framework should wait for
|
||||
* processors to stop running before calling the {@link nifi.processor.annotation.OnUnscheduled @OnUnscheduled} annotation
|
||||
*/
|
||||
void run(int iterations, boolean stopOnFinish, final boolean initialize, final long runWait);
|
||||
|
|
|
@ -35,5 +35,10 @@
|
|||
<artifactId>nifi-processor-utils</artifactId>
|
||||
<scope>test</scope>
|
||||
</dependency>
|
||||
<dependency>
|
||||
<groupId>commons-io</groupId>
|
||||
<artifactId>commons-io</artifactId>
|
||||
<scope>test</scope>
|
||||
</dependency>
|
||||
</dependencies>
|
||||
</project>
|
||||
|
|
|
@ -70,13 +70,12 @@ public class DocGenerator {
|
|||
|
||||
for (final Class<?> extensionClass : extensionClasses) {
|
||||
if (ConfigurableComponent.class.isAssignableFrom(extensionClass)) {
|
||||
final Class<? extends ConfigurableComponent> componentClass = extensionClass
|
||||
.asSubclass(ConfigurableComponent.class);
|
||||
final Class<? extends ConfigurableComponent> componentClass = extensionClass.asSubclass(ConfigurableComponent.class);
|
||||
try {
|
||||
logger.debug("Documenting: " + componentClass);
|
||||
document(explodedNiFiDocsDir, componentClass);
|
||||
} catch (Exception e) {
|
||||
logger.warn("Unable to document: " + componentClass);
|
||||
logger.warn("Unable to document: " + componentClass, e);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
@ -16,15 +16,21 @@
|
|||
*/
|
||||
package org.apache.nifi.documentation.init;
|
||||
|
||||
import org.apache.nifi.annotation.lifecycle.OnRemoved;
|
||||
import org.apache.nifi.annotation.lifecycle.OnShutdown;
|
||||
import org.apache.nifi.components.ConfigurableComponent;
|
||||
import org.apache.nifi.controller.ControllerService;
|
||||
import org.apache.nifi.documentation.ConfigurableComponentInitializer;
|
||||
import org.apache.nifi.documentation.mock.MockConfigurationContext;
|
||||
import org.apache.nifi.documentation.mock.MockControllerServiceInitializationContext;
|
||||
import org.apache.nifi.documentation.mock.MockProcessorLogger;
|
||||
import org.apache.nifi.documentation.util.ReflectionUtils;
|
||||
import org.apache.nifi.logging.ProcessorLog;
|
||||
import org.apache.nifi.nar.NarCloseable;
|
||||
import org.apache.nifi.reporting.InitializationException;
|
||||
|
||||
/**
|
||||
* Initializes a ControllerService using a
|
||||
* MockControllerServiceInitializationContext
|
||||
* Initializes a ControllerService using a MockControllerServiceInitializationContext
|
||||
*
|
||||
*
|
||||
*/
|
||||
|
@ -33,6 +39,14 @@ public class ControllerServiceInitializer implements ConfigurableComponentInitia
|
|||
@Override
|
||||
public void initialize(ConfigurableComponent component) throws InitializationException {
|
||||
ControllerService controllerService = (ControllerService) component;
|
||||
controllerService.initialize(new MockControllerServiceInitializationContext());
|
||||
|
||||
try (NarCloseable narCloseable = NarCloseable.withNarLoader()) {
|
||||
controllerService.initialize(new MockControllerServiceInitializationContext());
|
||||
|
||||
final ProcessorLog logger = new MockProcessorLogger();
|
||||
final MockConfigurationContext context = new MockConfigurationContext();
|
||||
ReflectionUtils.quietlyInvokeMethodsWithAnnotations(OnRemoved.class, null, controllerService, logger, context);
|
||||
ReflectionUtils.quietlyInvokeMethodsWithAnnotations(OnShutdown.class, null, controllerService, logger, context);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
@ -16,9 +16,16 @@
|
|||
*/
|
||||
package org.apache.nifi.documentation.init;
|
||||
|
||||
import org.apache.nifi.annotation.lifecycle.OnRemoved;
|
||||
import org.apache.nifi.annotation.lifecycle.OnShutdown;
|
||||
import org.apache.nifi.components.ConfigurableComponent;
|
||||
import org.apache.nifi.documentation.ConfigurableComponentInitializer;
|
||||
import org.apache.nifi.documentation.mock.MockProcessContext;
|
||||
import org.apache.nifi.documentation.mock.MockProcessorInitializationContext;
|
||||
import org.apache.nifi.documentation.mock.MockProcessorLogger;
|
||||
import org.apache.nifi.documentation.util.ReflectionUtils;
|
||||
import org.apache.nifi.logging.ProcessorLog;
|
||||
import org.apache.nifi.nar.NarCloseable;
|
||||
import org.apache.nifi.processor.Processor;
|
||||
|
||||
/**
|
||||
|
@ -31,7 +38,13 @@ public class ProcessorInitializer implements ConfigurableComponentInitializer {
|
|||
@Override
|
||||
public void initialize(ConfigurableComponent component) {
|
||||
Processor processor = (Processor) component;
|
||||
processor.initialize(new MockProcessorInitializationContext());
|
||||
}
|
||||
try (NarCloseable narCloseable = NarCloseable.withNarLoader()) {
|
||||
processor.initialize(new MockProcessorInitializationContext());
|
||||
|
||||
final ProcessorLog logger = new MockProcessorLogger();
|
||||
final MockProcessContext context = new MockProcessContext();
|
||||
ReflectionUtils.quietlyInvokeMethodsWithAnnotations(OnRemoved.class, null, processor, logger, context);
|
||||
ReflectionUtils.quietlyInvokeMethodsWithAnnotations(OnShutdown.class, null, processor, logger, context);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
@ -16,9 +16,15 @@
|
|||
*/
|
||||
package org.apache.nifi.documentation.init;
|
||||
|
||||
import org.apache.nifi.annotation.lifecycle.OnRemoved;
|
||||
import org.apache.nifi.annotation.lifecycle.OnShutdown;
|
||||
import org.apache.nifi.components.ConfigurableComponent;
|
||||
import org.apache.nifi.documentation.ConfigurableComponentInitializer;
|
||||
import org.apache.nifi.documentation.mock.MockConfigurationContext;
|
||||
import org.apache.nifi.documentation.mock.MockProcessorLogger;
|
||||
import org.apache.nifi.documentation.mock.MockReportingInitializationContext;
|
||||
import org.apache.nifi.documentation.util.ReflectionUtils;
|
||||
import org.apache.nifi.nar.NarCloseable;
|
||||
import org.apache.nifi.reporting.InitializationException;
|
||||
import org.apache.nifi.reporting.ReportingTask;
|
||||
|
||||
|
@ -32,6 +38,12 @@ public class ReportingTaskingInitializer implements ConfigurableComponentInitial
|
|||
@Override
|
||||
public void initialize(ConfigurableComponent component) throws InitializationException {
|
||||
ReportingTask reportingTask = (ReportingTask) component;
|
||||
reportingTask.initialize(new MockReportingInitializationContext());
|
||||
try (NarCloseable narCloseable = NarCloseable.withNarLoader()) {
|
||||
reportingTask.initialize(new MockReportingInitializationContext());
|
||||
|
||||
final MockConfigurationContext context = new MockConfigurationContext();
|
||||
ReflectionUtils.quietlyInvokeMethodsWithAnnotations(OnRemoved.class, null, reportingTask, new MockProcessorLogger(), context);
|
||||
ReflectionUtils.quietlyInvokeMethodsWithAnnotations(OnShutdown.class, null, reportingTask, new MockProcessorLogger(), context);
|
||||
}
|
||||
}
|
||||
}
|
||||
|
|
|
@ -0,0 +1,38 @@
|
|||
/*
|
||||
* 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.documentation.mock;
|
||||
|
||||
import java.util.Collections;
|
||||
import java.util.Map;
|
||||
|
||||
import org.apache.nifi.components.PropertyDescriptor;
|
||||
import org.apache.nifi.components.PropertyValue;
|
||||
import org.apache.nifi.controller.ConfigurationContext;
|
||||
|
||||
public class MockConfigurationContext implements ConfigurationContext {
|
||||
|
||||
@Override
|
||||
public PropertyValue getProperty(PropertyDescriptor property) {
|
||||
return null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public Map<PropertyDescriptor, String> getProperties() {
|
||||
return Collections.emptyMap();
|
||||
}
|
||||
|
||||
}
|
|
@ -0,0 +1,85 @@
|
|||
/*
|
||||
* 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.documentation.mock;
|
||||
|
||||
import java.util.Collections;
|
||||
import java.util.Map;
|
||||
import java.util.Set;
|
||||
|
||||
import org.apache.nifi.components.PropertyDescriptor;
|
||||
import org.apache.nifi.components.PropertyValue;
|
||||
import org.apache.nifi.controller.ControllerServiceLookup;
|
||||
import org.apache.nifi.processor.ProcessContext;
|
||||
import org.apache.nifi.processor.Relationship;
|
||||
|
||||
public class MockProcessContext implements ProcessContext {
|
||||
|
||||
@Override
|
||||
public PropertyValue getProperty(PropertyDescriptor descriptor) {
|
||||
return null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public PropertyValue getProperty(String propertyName) {
|
||||
return null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public PropertyValue newPropertyValue(String rawValue) {
|
||||
return null;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void yield() {
|
||||
|
||||
}
|
||||
|
||||
@Override
|
||||
public int getMaxConcurrentTasks() {
|
||||
return 0;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String getAnnotationData() {
|
||||
return "";
|
||||
}
|
||||
|
||||
@Override
|
||||
public Map<PropertyDescriptor, String> getProperties() {
|
||||
return Collections.emptyMap();
|
||||
}
|
||||
|
||||
@Override
|
||||
public String encrypt(String unencrypted) {
|
||||
return unencrypted;
|
||||
}
|
||||
|
||||
@Override
|
||||
public String decrypt(String encrypted) {
|
||||
return encrypted;
|
||||
}
|
||||
|
||||
@Override
|
||||
public ControllerServiceLookup getControllerServiceLookup() {
|
||||
return new MockControllerServiceLookup();
|
||||
}
|
||||
|
||||
@Override
|
||||
public Set<Relationship> getAvailableRelationships() {
|
||||
return Collections.emptySet();
|
||||
}
|
||||
}
|
|
@ -0,0 +1,139 @@
|
|||
/*
|
||||
* 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.documentation.util;
|
||||
|
||||
import java.lang.annotation.Annotation;
|
||||
import java.lang.reflect.InvocationTargetException;
|
||||
import java.lang.reflect.Method;
|
||||
import java.util.ArrayList;
|
||||
import java.util.List;
|
||||
|
||||
import org.apache.nifi.logging.ProcessorLog;
|
||||
import org.slf4j.Logger;
|
||||
import org.slf4j.LoggerFactory;
|
||||
|
||||
/**
|
||||
* This class is a copy of org.apache.nifi.util.ReflectionUtils. Ultimately the documentation generation
|
||||
* component should be moved to a place where it can depend on this directly instead of copying it in.
|
||||
*
|
||||
*
|
||||
*/
|
||||
public class ReflectionUtils {
|
||||
|
||||
private final static Logger LOG = LoggerFactory.getLogger(ReflectionUtils.class);
|
||||
/**
|
||||
* Invokes all methods on the given instance that have been annotated with the given preferredAnnotation and if no such method exists will invoke all methods on the given instance that have been
|
||||
* annotated with the given alternateAnnotation, if any exists. If the signature of the method that is defined in <code>instance</code> uses 1 or more parameters, those parameters must be
|
||||
* specified by the <code>args</code> parameter. However, if more arguments are supplied by the <code>args</code> parameter than needed, the extra arguments will be ignored.
|
||||
*
|
||||
* @param preferredAnnotation preferred
|
||||
* @param alternateAnnotation alternate
|
||||
* @param instance instance
|
||||
* @param logger the ProcessorLog to use for logging any errors. If null, will use own logger, but that will not generate bulletins or easily tie to the Processor's log messages.
|
||||
* @param args args
|
||||
* @return <code>true</code> if all appropriate methods were invoked and returned without throwing an Exception, <code>false</code> if one of the methods threw an Exception or could not be
|
||||
* invoked; if <code>false</code> is returned, an error will have been logged.
|
||||
*/
|
||||
public static boolean quietlyInvokeMethodsWithAnnotations(
|
||||
final Class<? extends Annotation> preferredAnnotation, final Class<? extends Annotation> alternateAnnotation, final Object instance, final ProcessorLog logger, final Object... args) {
|
||||
final List<Class<? extends Annotation>> annotationClasses = new ArrayList<>(alternateAnnotation == null ? 1 : 2);
|
||||
annotationClasses.add(preferredAnnotation);
|
||||
if (alternateAnnotation != null) {
|
||||
annotationClasses.add(alternateAnnotation);
|
||||
}
|
||||
|
||||
boolean annotationFound = false;
|
||||
for (final Class<? extends Annotation> annotationClass : annotationClasses) {
|
||||
if (annotationFound) {
|
||||
break;
|
||||
}
|
||||
|
||||
for (final Method method : instance.getClass().getMethods()) {
|
||||
if (method.isAnnotationPresent(annotationClass)) {
|
||||
annotationFound = true;
|
||||
|
||||
final boolean isAccessible = method.isAccessible();
|
||||
method.setAccessible(true);
|
||||
|
||||
try {
|
||||
final Class<?>[] argumentTypes = method.getParameterTypes();
|
||||
if (argumentTypes.length > args.length) {
|
||||
if (logger == null) {
|
||||
LOG.error("Unable to invoke method {} on {} because method expects {} parameters but only {} were given",
|
||||
new Object[]{method.getName(), instance, argumentTypes.length, args.length});
|
||||
} else {
|
||||
logger.error("Unable to invoke method {} on {} because method expects {} parameters but only {} were given",
|
||||
new Object[]{method.getName(), instance, argumentTypes.length, args.length});
|
||||
}
|
||||
|
||||
return false;
|
||||
}
|
||||
|
||||
for (int i = 0; i < argumentTypes.length; i++) {
|
||||
final Class<?> argType = argumentTypes[i];
|
||||
if (!argType.isAssignableFrom(args[i].getClass())) {
|
||||
if (logger == null) {
|
||||
LOG.error("Unable to invoke method {} on {} because method parameter {} is expected to be of type {} but argument passed was of type {}",
|
||||
new Object[]{method.getName(), instance, i, argType, args[i].getClass()});
|
||||
} else {
|
||||
logger.error("Unable to invoke method {} on {} because method parameter {} is expected to be of type {} but argument passed was of type {}",
|
||||
new Object[]{method.getName(), instance, i, argType, args[i].getClass()});
|
||||
}
|
||||
|
||||
return false;
|
||||
}
|
||||
}
|
||||
|
||||
try {
|
||||
if (argumentTypes.length == args.length) {
|
||||
method.invoke(instance, args);
|
||||
} else {
|
||||
final Object[] argsToPass = new Object[argumentTypes.length];
|
||||
for (int i = 0; i < argsToPass.length; i++) {
|
||||
argsToPass[i] = args[i];
|
||||
}
|
||||
|
||||
method.invoke(instance, argsToPass);
|
||||
}
|
||||
} catch (final InvocationTargetException ite) {
|
||||
if (logger == null) {
|
||||
LOG.error("Unable to invoke method {} on {} due to {}", new Object[]{method.getName(), instance, ite.getCause()});
|
||||
LOG.error("", ite.getCause());
|
||||
} else {
|
||||
logger.error("Unable to invoke method {} on {} due to {}", new Object[]{method.getName(), instance, ite.getCause()});
|
||||
}
|
||||
} catch (final IllegalAccessException | IllegalArgumentException t) {
|
||||
if (logger == null) {
|
||||
LOG.error("Unable to invoke method {} on {} due to {}", new Object[]{method.getName(), instance, t});
|
||||
LOG.error("", t);
|
||||
} else {
|
||||
logger.error("Unable to invoke method {} on {} due to {}", new Object[]{method.getName(), instance, t});
|
||||
}
|
||||
|
||||
return false;
|
||||
}
|
||||
} finally {
|
||||
if (!isAccessible) {
|
||||
method.setAccessible(false);
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
}
|
||||
return true;
|
||||
}
|
||||
}
|
|
@ -0,0 +1,96 @@
|
|||
/*
|
||||
* 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.documentation;
|
||||
|
||||
import java.io.BufferedInputStream;
|
||||
import java.io.File;
|
||||
import java.io.FileInputStream;
|
||||
import java.io.IOException;
|
||||
import java.io.InputStream;
|
||||
|
||||
import org.apache.commons.io.FileUtils;
|
||||
import org.apache.nifi.nar.ExtensionManager;
|
||||
import org.apache.nifi.nar.NarClassLoaders;
|
||||
import org.apache.nifi.nar.NarUnpacker;
|
||||
import org.apache.nifi.util.NiFiProperties;
|
||||
import org.junit.Assert;
|
||||
import org.junit.Test;
|
||||
import org.junit.rules.TemporaryFolder;
|
||||
|
||||
public class DocGeneratorTest {
|
||||
|
||||
@Test
|
||||
public void testProcessorLoadsNarResources() throws IOException, ClassNotFoundException {
|
||||
TemporaryFolder temporaryFolder = new TemporaryFolder();
|
||||
temporaryFolder.create();
|
||||
|
||||
NiFiProperties properties = loadSpecifiedProperties("/conf/nifi.properties");
|
||||
properties.setProperty(NiFiProperties.COMPONENT_DOCS_DIRECTORY, temporaryFolder.getRoot().getAbsolutePath());
|
||||
|
||||
NarUnpacker.unpackNars(properties);
|
||||
|
||||
NarClassLoaders.load(properties);
|
||||
|
||||
ExtensionManager.discoverExtensions();
|
||||
|
||||
DocGenerator.generate(properties);
|
||||
|
||||
File processorDirectory = new File(temporaryFolder.getRoot(), "org.apache.nifi.processors.WriteResourceToStream");
|
||||
File indexHtml = new File(processorDirectory, "index.html");
|
||||
Assert.assertTrue(indexHtml + " should have been generated", indexHtml.exists());
|
||||
String generatedHtml = FileUtils.readFileToString(indexHtml);
|
||||
Assert.assertNotNull(generatedHtml);
|
||||
Assert.assertTrue(generatedHtml.contains("This example processor loads a resource from the nar and writes it to the FlowFile content"));
|
||||
Assert.assertTrue(generatedHtml.contains("files that were successfully processed"));
|
||||
Assert.assertTrue(generatedHtml.contains("files that were not successfully processed"));
|
||||
Assert.assertTrue(generatedHtml.contains("resources"));
|
||||
}
|
||||
|
||||
private NiFiProperties loadSpecifiedProperties(String propertiesFile) {
|
||||
String file = DocGeneratorTest.class.getResource(propertiesFile).getFile();
|
||||
|
||||
System.setProperty(NiFiProperties.PROPERTIES_FILE_PATH, file);
|
||||
|
||||
NiFiProperties properties = NiFiProperties.getInstance();
|
||||
|
||||
// clear out existing properties
|
||||
for (String prop : properties.stringPropertyNames()) {
|
||||
properties.remove(prop);
|
||||
}
|
||||
|
||||
InputStream inStream = null;
|
||||
try {
|
||||
inStream = new BufferedInputStream(new FileInputStream(file));
|
||||
properties.load(inStream);
|
||||
} catch (final Exception ex) {
|
||||
throw new RuntimeException("Cannot load properties file due to "
|
||||
+ ex.getLocalizedMessage(), ex);
|
||||
} finally {
|
||||
if (null != inStream) {
|
||||
try {
|
||||
inStream.close();
|
||||
} catch (final Exception ex) {
|
||||
/**
|
||||
* do nothing *
|
||||
*/
|
||||
}
|
||||
}
|
||||
}
|
||||
|
||||
return properties;
|
||||
}
|
||||
}
|
|
@ -22,26 +22,32 @@ import java.util.List;
|
|||
|
||||
import org.apache.nifi.annotation.documentation.CapabilityDescription;
|
||||
import org.apache.nifi.annotation.documentation.Tags;
|
||||
import org.apache.nifi.annotation.lifecycle.OnRemoved;
|
||||
import org.apache.nifi.annotation.lifecycle.OnShutdown;
|
||||
import org.apache.nifi.components.PropertyDescriptor;
|
||||
import org.apache.nifi.controller.AbstractControllerService;
|
||||
import org.apache.nifi.controller.ConfigurationContext;
|
||||
import org.apache.nifi.processor.util.StandardValidators;
|
||||
|
||||
@CapabilityDescription("A documented controller service that can help you do things")
|
||||
@Tags({"one", "two", "three"})
|
||||
public class FullyDocumentedControllerService extends AbstractControllerService implements SampleService{
|
||||
@Tags({ "one", "two", "three" })
|
||||
public class FullyDocumentedControllerService extends AbstractControllerService implements SampleService {
|
||||
|
||||
public static final PropertyDescriptor KEYSTORE = new PropertyDescriptor.Builder().name("Keystore Filename")
|
||||
.description("The fully-qualified filename of the Keystore").defaultValue(null)
|
||||
public static final PropertyDescriptor KEYSTORE = new PropertyDescriptor.Builder().name("Keystore Filename").description("The fully-qualified filename of the Keystore").defaultValue(null)
|
||||
.addValidator(StandardValidators.FILE_EXISTS_VALIDATOR).sensitive(false).build();
|
||||
public static final PropertyDescriptor KEYSTORE_TYPE = new PropertyDescriptor.Builder().name("Keystore Type")
|
||||
.description("The Type of the Keystore").allowableValues("JKS", "PKCS12")
|
||||
public static final PropertyDescriptor KEYSTORE_TYPE = new PropertyDescriptor.Builder().name("Keystore Type").description("The Type of the Keystore").allowableValues("JKS", "PKCS12")
|
||||
.addValidator(StandardValidators.NON_EMPTY_VALIDATOR).defaultValue("JKS").sensitive(false).build();
|
||||
public static final PropertyDescriptor KEYSTORE_PASSWORD = new PropertyDescriptor.Builder()
|
||||
.name("Keystore Password").defaultValue(null).description("The password for the Keystore")
|
||||
public static final PropertyDescriptor KEYSTORE_PASSWORD = new PropertyDescriptor.Builder().name("Keystore Password").defaultValue(null).description("The password for the Keystore")
|
||||
.addValidator(StandardValidators.NON_EMPTY_VALIDATOR).sensitive(true).build();
|
||||
|
||||
private static final List<PropertyDescriptor> properties;
|
||||
|
||||
private int onRemovedNoArgs = 0;
|
||||
private int onRemovedArgs = 0;
|
||||
|
||||
private int onShutdownNoArgs = 0;
|
||||
private int onShutdownArgs = 0;
|
||||
|
||||
static {
|
||||
List<PropertyDescriptor> props = new ArrayList<>();
|
||||
props.add(KEYSTORE);
|
||||
|
@ -57,6 +63,41 @@ public class FullyDocumentedControllerService extends AbstractControllerService
|
|||
|
||||
@Override
|
||||
public void doSomething() {
|
||||
// TODO Auto-generated method stub
|
||||
}
|
||||
|
||||
@OnRemoved
|
||||
public void onRemovedNoArgs() {
|
||||
onRemovedNoArgs++;
|
||||
}
|
||||
|
||||
@OnRemoved
|
||||
public void onRemovedArgs(ConfigurationContext context) {
|
||||
onRemovedArgs++;
|
||||
}
|
||||
|
||||
@OnShutdown
|
||||
public void onShutdownNoArgs() {
|
||||
onShutdownNoArgs++;
|
||||
}
|
||||
|
||||
@OnShutdown
|
||||
public void onShutdownArgs(ConfigurationContext context) {
|
||||
onShutdownArgs++;
|
||||
}
|
||||
|
||||
public int getOnRemovedNoArgs() {
|
||||
return onRemovedNoArgs;
|
||||
}
|
||||
|
||||
public int getOnRemovedArgs() {
|
||||
return onRemovedArgs;
|
||||
}
|
||||
|
||||
public int getOnShutdownNoArgs() {
|
||||
return onShutdownNoArgs;
|
||||
}
|
||||
|
||||
public int getOnShutdownArgs() {
|
||||
return onShutdownArgs;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -23,13 +23,15 @@ import java.util.List;
|
|||
import java.util.Set;
|
||||
|
||||
import org.apache.nifi.annotation.behavior.DynamicProperty;
|
||||
import org.apache.nifi.annotation.documentation.CapabilityDescription;
|
||||
import org.apache.nifi.annotation.behavior.DynamicRelationship;
|
||||
import org.apache.nifi.annotation.behavior.ReadsAttribute;
|
||||
import org.apache.nifi.annotation.documentation.SeeAlso;
|
||||
import org.apache.nifi.annotation.documentation.Tags;
|
||||
import org.apache.nifi.annotation.behavior.WritesAttribute;
|
||||
import org.apache.nifi.annotation.behavior.WritesAttributes;
|
||||
import org.apache.nifi.annotation.documentation.CapabilityDescription;
|
||||
import org.apache.nifi.annotation.documentation.SeeAlso;
|
||||
import org.apache.nifi.annotation.documentation.Tags;
|
||||
import org.apache.nifi.annotation.lifecycle.OnRemoved;
|
||||
import org.apache.nifi.annotation.lifecycle.OnShutdown;
|
||||
import org.apache.nifi.components.AllowableValue;
|
||||
import org.apache.nifi.components.PropertyDescriptor;
|
||||
import org.apache.nifi.processor.AbstractProcessor;
|
||||
|
@ -88,6 +90,12 @@ public class FullyDocumentedProcessor extends AbstractProcessor {
|
|||
private List<PropertyDescriptor> properties;
|
||||
private Set<Relationship> relationships;
|
||||
|
||||
private int onRemovedNoArgs = 0;
|
||||
private int onRemovedArgs = 0;
|
||||
|
||||
private int onShutdownNoArgs = 0;
|
||||
private int onShutdownArgs = 0;
|
||||
|
||||
@Override
|
||||
protected void init(ProcessorInitializationContext context) {
|
||||
final List<PropertyDescriptor> properties = new ArrayList<>();
|
||||
|
@ -126,4 +134,39 @@ public class FullyDocumentedProcessor extends AbstractProcessor {
|
|||
.description("This is a property you can use or not").dynamic(true).build();
|
||||
}
|
||||
|
||||
@OnRemoved
|
||||
public void onRemovedNoArgs() {
|
||||
onRemovedNoArgs++;
|
||||
}
|
||||
|
||||
@OnRemoved
|
||||
public void onRemovedArgs(ProcessContext context) {
|
||||
onRemovedArgs++;
|
||||
}
|
||||
|
||||
@OnShutdown
|
||||
public void onShutdownNoArgs() {
|
||||
onShutdownNoArgs++;
|
||||
}
|
||||
|
||||
@OnShutdown
|
||||
public void onShutdownArgs(ProcessContext context) {
|
||||
onShutdownArgs++;
|
||||
}
|
||||
|
||||
public int getOnRemovedNoArgs() {
|
||||
return onRemovedNoArgs;
|
||||
}
|
||||
|
||||
public int getOnRemovedArgs() {
|
||||
return onRemovedArgs;
|
||||
}
|
||||
|
||||
public int getOnShutdownNoArgs() {
|
||||
return onShutdownNoArgs;
|
||||
}
|
||||
|
||||
public int getOnShutdownArgs() {
|
||||
return onShutdownArgs;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -20,8 +20,11 @@ import java.util.ArrayList;
|
|||
import java.util.List;
|
||||
|
||||
import org.apache.nifi.components.PropertyDescriptor;
|
||||
import org.apache.nifi.controller.ConfigurationContext;
|
||||
import org.apache.nifi.annotation.documentation.CapabilityDescription;
|
||||
import org.apache.nifi.annotation.documentation.Tags;
|
||||
import org.apache.nifi.annotation.lifecycle.OnRemoved;
|
||||
import org.apache.nifi.annotation.lifecycle.OnShutdown;
|
||||
import org.apache.nifi.reporting.AbstractReportingTask;
|
||||
import org.apache.nifi.reporting.ReportingContext;
|
||||
|
||||
|
@ -35,6 +38,12 @@ public class FullyDocumentedReportingTask extends AbstractReportingTask {
|
|||
"Specifies whether or not to show the difference in values between the current status and the previous status")
|
||||
.required(true).allowableValues("true", "false").defaultValue("true").build();
|
||||
|
||||
private int onRemovedNoArgs = 0;
|
||||
private int onRemovedArgs = 0;
|
||||
|
||||
private int onShutdownNoArgs = 0;
|
||||
private int onShutdownArgs = 0;
|
||||
|
||||
@Override
|
||||
public final List<PropertyDescriptor> getSupportedPropertyDescriptors() {
|
||||
final List<PropertyDescriptor> descriptors = new ArrayList<>();
|
||||
|
@ -44,7 +53,42 @@ public class FullyDocumentedReportingTask extends AbstractReportingTask {
|
|||
|
||||
@Override
|
||||
public void onTrigger(ReportingContext context) {
|
||||
// TODO Auto-generated method stub
|
||||
|
||||
}
|
||||
|
||||
@OnRemoved
|
||||
public void onRemovedNoArgs() {
|
||||
onRemovedNoArgs++;
|
||||
}
|
||||
|
||||
@OnRemoved
|
||||
public void onRemovedArgs(ConfigurationContext context) {
|
||||
onRemovedArgs++;
|
||||
}
|
||||
|
||||
@OnShutdown
|
||||
public void onShutdownNoArgs() {
|
||||
onShutdownNoArgs++;
|
||||
}
|
||||
|
||||
@OnShutdown
|
||||
public void onShutdownArgs(ConfigurationContext context) {
|
||||
onShutdownArgs++;
|
||||
}
|
||||
|
||||
public int getOnRemovedNoArgs() {
|
||||
return onRemovedNoArgs;
|
||||
}
|
||||
|
||||
public int getOnRemovedArgs() {
|
||||
return onRemovedArgs;
|
||||
}
|
||||
|
||||
public int getOnShutdownNoArgs() {
|
||||
return onShutdownNoArgs;
|
||||
}
|
||||
|
||||
public int getOnShutdownArgs() {
|
||||
return onShutdownArgs;
|
||||
}
|
||||
}
|
||||
|
|
|
@ -28,26 +28,30 @@ import org.apache.nifi.documentation.example.ControllerServiceWithLogger;
|
|||
import org.apache.nifi.documentation.example.FullyDocumentedControllerService;
|
||||
import org.apache.nifi.documentation.example.FullyDocumentedReportingTask;
|
||||
import org.apache.nifi.documentation.example.ReportingTaskWithLogger;
|
||||
import org.apache.nifi.documentation.init.ControllerServiceInitializer;
|
||||
import org.apache.nifi.documentation.init.ReportingTaskingInitializer;
|
||||
import org.apache.nifi.documentation.mock.MockControllerServiceInitializationContext;
|
||||
import org.apache.nifi.documentation.mock.MockReportingInitializationContext;
|
||||
import org.apache.nifi.reporting.InitializationException;
|
||||
import org.apache.nifi.reporting.ReportingTask;
|
||||
import org.junit.Assert;
|
||||
import org.junit.Test;
|
||||
|
||||
public class HtmlDocumentationWriterTest {
|
||||
|
||||
@Test
|
||||
public void testJoin() {
|
||||
assertEquals("a, b, c", HtmlDocumentationWriter.join(new String[]{"a", "b", "c"}, ", "));
|
||||
assertEquals("a, b", HtmlDocumentationWriter.join(new String[]{"a", "b"}, ", "));
|
||||
assertEquals("a", HtmlDocumentationWriter.join(new String[]{"a"}, ", "));
|
||||
assertEquals("a, b, c", HtmlDocumentationWriter.join(new String[] { "a", "b", "c" }, ", "));
|
||||
assertEquals("a, b", HtmlDocumentationWriter.join(new String[] { "a", "b" }, ", "));
|
||||
assertEquals("a", HtmlDocumentationWriter.join(new String[] { "a" }, ", "));
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testDocumentControllerService() throws InitializationException, IOException {
|
||||
|
||||
ControllerService controllerService = new FullyDocumentedControllerService();
|
||||
controllerService.initialize(new MockControllerServiceInitializationContext());
|
||||
FullyDocumentedControllerService controllerService = new FullyDocumentedControllerService();
|
||||
ControllerServiceInitializer initializer = new ControllerServiceInitializer();
|
||||
initializer.initialize(controllerService);
|
||||
|
||||
DocumentationWriter writer = new HtmlDocumentationWriter();
|
||||
|
||||
|
@ -71,13 +75,21 @@ public class HtmlDocumentationWriterTest {
|
|||
assertContains(results, "JKS");
|
||||
assertContains(results, "PKCS12");
|
||||
assertContains(results, "Sensitive Property: true");
|
||||
|
||||
// verify the right OnRemoved and OnShutdown methods were called
|
||||
Assert.assertEquals(1, controllerService.getOnRemovedArgs());
|
||||
Assert.assertEquals(1, controllerService.getOnRemovedNoArgs());
|
||||
|
||||
Assert.assertEquals(1, controllerService.getOnShutdownArgs());
|
||||
Assert.assertEquals(1, controllerService.getOnShutdownNoArgs());
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testDocumentReportingTask() throws InitializationException, IOException {
|
||||
|
||||
ReportingTask reportingTask = new FullyDocumentedReportingTask();
|
||||
reportingTask.initialize(new MockReportingInitializationContext());
|
||||
FullyDocumentedReportingTask reportingTask = new FullyDocumentedReportingTask();
|
||||
ReportingTaskingInitializer initializer = new ReportingTaskingInitializer();
|
||||
initializer.initialize(reportingTask);
|
||||
|
||||
DocumentationWriter writer = new HtmlDocumentationWriter();
|
||||
|
||||
|
@ -99,6 +111,13 @@ public class HtmlDocumentationWriterTest {
|
|||
assertContains(results, "Specifies whether or not to show the difference in values between the current status and the previous status");
|
||||
assertContains(results, "true");
|
||||
assertContains(results, "false");
|
||||
|
||||
// verify the right OnRemoved and OnShutdown methods were called
|
||||
Assert.assertEquals(1, reportingTask.getOnRemovedArgs());
|
||||
Assert.assertEquals(1, reportingTask.getOnRemovedNoArgs());
|
||||
|
||||
Assert.assertEquals(1, reportingTask.getOnShutdownArgs());
|
||||
Assert.assertEquals(1, reportingTask.getOnShutdownNoArgs());
|
||||
}
|
||||
|
||||
@Test
|
||||
|
|
|
@ -27,7 +27,8 @@ import org.apache.nifi.documentation.DocumentationWriter;
|
|||
import org.apache.nifi.documentation.example.FullyDocumentedProcessor;
|
||||
import org.apache.nifi.documentation.example.NakedProcessor;
|
||||
import org.apache.nifi.documentation.example.ProcessorWithLogger;
|
||||
import org.apache.nifi.documentation.mock.MockProcessorInitializationContext;
|
||||
import org.apache.nifi.documentation.init.ProcessorInitializer;
|
||||
import org.junit.Assert;
|
||||
import org.junit.Test;
|
||||
|
||||
public class ProcessorDocumentationWriterTest {
|
||||
|
@ -35,7 +36,8 @@ public class ProcessorDocumentationWriterTest {
|
|||
@Test
|
||||
public void testFullyDocumentedProcessor() throws IOException {
|
||||
FullyDocumentedProcessor processor = new FullyDocumentedProcessor();
|
||||
processor.initialize(new MockProcessorInitializationContext());
|
||||
ProcessorInitializer initializer = new ProcessorInitializer();
|
||||
initializer.initialize(processor);
|
||||
|
||||
DocumentationWriter writer = new HtmlProcessorDocumentationWriter();
|
||||
|
||||
|
@ -70,12 +72,20 @@ public class ProcessorDocumentationWriterTest {
|
|||
assertNotContains(results, "No description provided.");
|
||||
assertNotContains(results, "No Tags provided.");
|
||||
assertNotContains(results, "Additional Details...");
|
||||
|
||||
// verify the right OnRemoved and OnShutdown methods were called
|
||||
Assert.assertEquals(1, processor.getOnRemovedArgs());
|
||||
Assert.assertEquals(1, processor.getOnRemovedNoArgs());
|
||||
|
||||
Assert.assertEquals(1, processor.getOnShutdownArgs());
|
||||
Assert.assertEquals(1, processor.getOnShutdownNoArgs());
|
||||
}
|
||||
|
||||
@Test
|
||||
public void testNakedProcessor() throws IOException {
|
||||
NakedProcessor processor = new NakedProcessor();
|
||||
processor.initialize(new MockProcessorInitializationContext());
|
||||
ProcessorInitializer initializer = new ProcessorInitializer();
|
||||
initializer.initialize(processor);
|
||||
|
||||
DocumentationWriter writer = new HtmlProcessorDocumentationWriter();
|
||||
|
||||
|
@ -103,7 +113,8 @@ public class ProcessorDocumentationWriterTest {
|
|||
@Test
|
||||
public void testProcessorWithLoggerInitialization() throws IOException {
|
||||
ProcessorWithLogger processor = new ProcessorWithLogger();
|
||||
processor.initialize(new MockProcessorInitializationContext());
|
||||
ProcessorInitializer initializer = new ProcessorInitializer();
|
||||
initializer.initialize(processor);
|
||||
|
||||
DocumentationWriter writer = new HtmlProcessorDocumentationWriter();
|
||||
|
||||
|
|
|
@ -0,0 +1,129 @@
|
|||
# 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.
|
||||
|
||||
# Core Properties #
|
||||
nifi.version=nifi-test 3.0.0
|
||||
nifi.flow.configuration.file=./target/flow.xml.gz
|
||||
nifi.flow.configuration.archive.dir=./target/archive/
|
||||
nifi.flowcontroller.autoResumeState=true
|
||||
nifi.flowcontroller.graceful.shutdown.period=10 sec
|
||||
nifi.flowservice.writedelay.interval=2 sec
|
||||
nifi.administrative.yield.duration=30 sec
|
||||
|
||||
nifi.reporting.task.configuration.file=./target/reporting-tasks.xml
|
||||
nifi.controller.service.configuration.file=./target/controller-services.xml
|
||||
nifi.templates.directory=./target/templates
|
||||
nifi.ui.banner.text=UI Banner Text
|
||||
nifi.ui.autorefresh.interval=30 sec
|
||||
nifi.nar.library.directory=./target/test-classes/lib/
|
||||
nifi.nar.library.directory.alt=./target/test-classes/lib2/
|
||||
|
||||
nifi.nar.working.directory=./target/work/nar/
|
||||
|
||||
# H2 Settings
|
||||
nifi.database.directory=./target/database_repository
|
||||
nifi.h2.url.append=;LOCK_TIMEOUT=25000;WRITE_DELAY=0;AUTO_SERVER=FALSE
|
||||
|
||||
# FlowFile Repository
|
||||
nifi.flowfile.repository.directory=./target/test-repo
|
||||
nifi.flowfile.repository.partitions=1
|
||||
nifi.flowfile.repository.checkpoint.interval=2 mins
|
||||
nifi.queue.swap.threshold=20000
|
||||
nifi.swap.storage.directory=./target/test-repo/swap
|
||||
nifi.swap.in.period=5 sec
|
||||
nifi.swap.in.threads=1
|
||||
nifi.swap.out.period=5 sec
|
||||
nifi.swap.out.threads=4
|
||||
|
||||
# Content Repository
|
||||
nifi.content.claim.max.appendable.size=10 MB
|
||||
nifi.content.claim.max.flow.files=100
|
||||
nifi.content.repository.directory.default=./target/content_repository
|
||||
|
||||
# Provenance Repository Properties
|
||||
nifi.provenance.repository.storage.directory=./target/provenance_repository
|
||||
nifi.provenance.repository.max.storage.time=24 hours
|
||||
nifi.provenance.repository.max.storage.size=1 GB
|
||||
nifi.provenance.repository.rollover.time=5 mins
|
||||
nifi.provenance.repository.rollover.size=100 MB
|
||||
|
||||
# Site to Site properties
|
||||
nifi.remote.input.socket.port=9990
|
||||
nifi.remote.input.secure=true
|
||||
|
||||
# web properties #
|
||||
nifi.web.war.directory=./target/lib
|
||||
nifi.web.http.host=
|
||||
nifi.web.http.port=8080
|
||||
nifi.web.https.host=
|
||||
nifi.web.https.port=
|
||||
nifi.web.jetty.working.directory=./target/work/jetty
|
||||
|
||||
# security properties #
|
||||
nifi.sensitive.props.key=key
|
||||
nifi.sensitive.props.algorithm=PBEWITHMD5AND256BITAES-CBC-OPENSSL
|
||||
nifi.sensitive.props.provider=BC
|
||||
|
||||
nifi.security.keystore=
|
||||
nifi.security.keystoreType=
|
||||
nifi.security.keystorePasswd=
|
||||
nifi.security.keyPasswd=
|
||||
nifi.security.truststore=
|
||||
nifi.security.truststoreType=
|
||||
nifi.security.truststorePasswd=
|
||||
nifi.security.needClientAuth=
|
||||
nifi.security.authorizedUsers.file=./target/conf/authorized-users.xml
|
||||
nifi.security.user.credential.cache.duration=24 hours
|
||||
nifi.security.user.authority.provider=nifi.authorization.FileAuthorizationProvider
|
||||
nifi.security.support.new.account.requests=
|
||||
nifi.security.default.user.roles=
|
||||
|
||||
# cluster common properties (cluster manager and nodes must have same values) #
|
||||
nifi.cluster.protocol.heartbeat.interval=5 sec
|
||||
nifi.cluster.protocol.is.secure=false
|
||||
nifi.cluster.protocol.socket.timeout=30 sec
|
||||
nifi.cluster.protocol.connection.handshake.timeout=45 sec
|
||||
# if multicast is used, then nifi.cluster.protocol.multicast.xxx properties must be configured #
|
||||
nifi.cluster.protocol.use.multicast=false
|
||||
nifi.cluster.protocol.multicast.address=
|
||||
nifi.cluster.protocol.multicast.port=
|
||||
nifi.cluster.protocol.multicast.service.broadcast.delay=500 ms
|
||||
nifi.cluster.protocol.multicast.service.locator.attempts=3
|
||||
nifi.cluster.protocol.multicast.service.locator.attempts.delay=1 sec
|
||||
|
||||
# cluster node properties (only configure for cluster nodes) #
|
||||
nifi.cluster.is.node=false
|
||||
nifi.cluster.node.address=
|
||||
nifi.cluster.node.protocol.port=
|
||||
nifi.cluster.node.protocol.threads=2
|
||||
# if multicast is not used, nifi.cluster.node.unicast.xxx must have same values as nifi.cluster.manager.xxx #
|
||||
nifi.cluster.node.unicast.manager.address=
|
||||
nifi.cluster.node.unicast.manager.protocol.port=
|
||||
nifi.cluster.node.unicast.manager.authority.provider.port=
|
||||
|
||||
# cluster manager properties (only configure for cluster manager) #
|
||||
nifi.cluster.is.manager=false
|
||||
nifi.cluster.manager.address=
|
||||
nifi.cluster.manager.protocol.port=
|
||||
nifi.cluster.manager.authority.provider.port=
|
||||
nifi.cluster.manager.authority.provider.threads=10
|
||||
nifi.cluster.manager.node.firewall.file=
|
||||
nifi.cluster.manager.node.event.history.size=10
|
||||
nifi.cluster.manager.node.api.connection.timeout=30 sec
|
||||
nifi.cluster.manager.node.api.read.timeout=30 sec
|
||||
nifi.cluster.manager.node.api.request.threads=10
|
||||
nifi.cluster.manager.flow.retrieval.delay=5 sec
|
||||
nifi.cluster.manager.protocol.threads=10
|
||||
nifi.cluster.manager.safemode.duration=0 sec
|
Binary file not shown.
Binary file not shown.
Binary file not shown.
|
@ -188,7 +188,7 @@ public class DataFlowDaoImpl implements DataFlowDao {
|
|||
return;
|
||||
}
|
||||
|
||||
if ((primaryEntry == null && restoreEntry != null) || (primaryEntry != null && restoreEntry == null)) {
|
||||
if (primaryEntry == null && restoreEntry != null || primaryEntry != null && restoreEntry == null) {
|
||||
throw new IllegalStateException(String.format("Primary file '%s' is different than restore file '%s'",
|
||||
primaryFile.getAbsoluteFile(), restoreFile.getAbsolutePath()));
|
||||
}
|
||||
|
@ -352,7 +352,7 @@ public class DataFlowDaoImpl implements DataFlowDao {
|
|||
final File[] files = dir.listFiles(new FilenameFilter() {
|
||||
@Override
|
||||
public boolean accept(File dir, String name) {
|
||||
return (name.equals(FLOW_PACKAGE) || name.endsWith(STALE_EXT) || name.endsWith(UNKNOWN_EXT));
|
||||
return name.equals(FLOW_PACKAGE) || name.endsWith(STALE_EXT) || name.endsWith(UNKNOWN_EXT);
|
||||
}
|
||||
});
|
||||
|
||||
|
@ -515,19 +515,10 @@ public class DataFlowDaoImpl implements DataFlowDao {
|
|||
final StandardDataFlow dataFlow = new StandardDataFlow(flowBytes, templateBytes, snippetBytes);
|
||||
dataFlow.setAutoStartProcessors(autoStart);
|
||||
|
||||
return new ClusterDataFlow(dataFlow, (clusterMetadata == null) ? null : clusterMetadata.getPrimaryNodeId(), controllerServiceBytes, reportingTaskBytes);
|
||||
return new ClusterDataFlow(dataFlow, clusterMetadata == null ? null : clusterMetadata.getPrimaryNodeId(), controllerServiceBytes, reportingTaskBytes);
|
||||
}
|
||||
|
||||
private void writeDataFlow(final File file, final ClusterDataFlow clusterDataFlow) throws IOException, JAXBException {
|
||||
|
||||
// get the data flow
|
||||
DataFlow dataFlow = clusterDataFlow.getDataFlow();
|
||||
|
||||
// if no dataflow, then write a new dataflow
|
||||
if (dataFlow == null) {
|
||||
dataFlow = new StandardDataFlow(new byte[0], new byte[0], new byte[0]);
|
||||
}
|
||||
|
||||
// setup the cluster metadata
|
||||
final ClusterMetadata clusterMetadata = new ClusterMetadata();
|
||||
clusterMetadata.setPrimaryNodeId(clusterDataFlow.getPrimaryNodeId());
|
||||
|
|
|
@ -130,7 +130,6 @@ public class FileSystemRepository implements ContentRepository {
|
|||
final String maxArchiveRetentionPeriod = properties.getProperty(NiFiProperties.CONTENT_ARCHIVE_MAX_RETENTION_PERIOD);
|
||||
final String maxArchiveSize = properties.getProperty(NiFiProperties.CONTENT_ARCHIVE_MAX_USAGE_PERCENTAGE);
|
||||
final String archiveBackPressureSize = properties.getProperty(NiFiProperties.CONTENT_ARCHIVE_BACK_PRESSURE_PERCENTAGE);
|
||||
final String archiveCleanupFrequency = properties.getProperty(NiFiProperties.CONTENT_ARCHIVE_CLEANUP_FREQUENCY);
|
||||
|
||||
if ("true".equalsIgnoreCase(enableArchiving)) {
|
||||
archiveData = true;
|
||||
|
@ -193,11 +192,23 @@ public class FileSystemRepository implements ContentRepository {
|
|||
LOG.info("Initializing FileSystemRepository with 'Always Sync' set to {}", alwaysSync);
|
||||
initializeRepository();
|
||||
|
||||
containerCleanupExecutor = new FlowEngine(containers.size(), "Cleanup FileSystemRepository Container", true);
|
||||
}
|
||||
|
||||
@Override
|
||||
public void initialize(final ContentClaimManager claimManager) {
|
||||
this.contentClaimManager = claimManager;
|
||||
|
||||
final NiFiProperties properties = NiFiProperties.getInstance();
|
||||
|
||||
final Map<String, Path> fileRespositoryPaths = properties.getContentRepositoryPaths();
|
||||
|
||||
executor.scheduleWithFixedDelay(new BinDestructableClaims(), 1, 1, TimeUnit.SECONDS);
|
||||
for (int i = 0; i < fileRespositoryPaths.size(); i++) {
|
||||
executor.scheduleWithFixedDelay(new ArchiveOrDestroyDestructableClaims(), 1, 1, TimeUnit.SECONDS);
|
||||
}
|
||||
|
||||
final String archiveCleanupFrequency = properties.getProperty(NiFiProperties.CONTENT_ARCHIVE_CLEANUP_FREQUENCY);
|
||||
final long cleanupMillis;
|
||||
if (archiveCleanupFrequency == null) {
|
||||
cleanupMillis = 1000L;
|
||||
|
@ -208,8 +219,6 @@ public class FileSystemRepository implements ContentRepository {
|
|||
throw new RuntimeException("Invalid value set for property " + NiFiProperties.CONTENT_ARCHIVE_CLEANUP_FREQUENCY);
|
||||
}
|
||||
}
|
||||
|
||||
containerCleanupExecutor = new FlowEngine(containers.size(), "Cleanup FileSystemRepository Container", true);
|
||||
for (final Map.Entry<String, Path> containerEntry : containers.entrySet()) {
|
||||
final String containerName = containerEntry.getKey();
|
||||
final Path containerPath = containerEntry.getValue();
|
||||
|
@ -218,11 +227,6 @@ public class FileSystemRepository implements ContentRepository {
|
|||
}
|
||||
}
|
||||
|
||||
@Override
|
||||
public void initialize(final ContentClaimManager claimManager) {
|
||||
this.contentClaimManager = claimManager;
|
||||
}
|
||||
|
||||
@Override
|
||||
public void shutdown() {
|
||||
executor.shutdown();
|
||||
|
|
|
@ -260,7 +260,7 @@ public final class StandardProcessSession implements ProcessSession, ProvenanceE
|
|||
|
||||
final FlowFileRecord clone = builder.build();
|
||||
final StandardRepositoryRecord newRecord = new StandardRepositoryRecord(destination.getFlowFileQueue());
|
||||
getProvenanceReporter().clone(currRec, clone);
|
||||
provenanceReporter.clone(currRec, clone, false);
|
||||
|
||||
final ContentClaim claim = clone.getContentClaim();
|
||||
if (claim != null) {
|
||||
|
|
Some files were not shown because too many files have changed in this diff Show More
Loading…
Reference in New Issue