This is an automated email from the ASF dual-hosted git repository. dwysakowicz pushed a commit to branch master in repository https://gitbox.apache.org/repos/asf/flink.git
The following commit(s) were added to refs/heads/master by this push: new 51210ee [FLINK-12402][table] Make validation error message in ResolveCallByArguments more user friendly 51210ee is described below commit 51210eed98d5553b40c13717597143491b0a0862 Author: Hequn Cheng <chenghe...@gmail.com> AuthorDate: Tue Jun 11 15:11:01 2019 +0800 [FLINK-12402][table] Make validation error message in ResolveCallByArguments more user friendly --- .../rules/ResolveCallByArgumentsRule.java | 35 +++++++++++++++------- .../table/validation/CalcValidationTest.scala | 13 ++++++++ .../validation/GroupWindowValidationTest.scala | 12 ++++++-- 3 files changed, 46 insertions(+), 14 deletions(-) diff --git a/flink-table/flink-table-planner/src/main/java/org/apache/flink/table/expressions/rules/ResolveCallByArgumentsRule.java b/flink-table/flink-table-planner/src/main/java/org/apache/flink/table/expressions/rules/ResolveCallByArgumentsRule.java index 7749b75..46c1834 100644 --- a/flink-table/flink-table-planner/src/main/java/org/apache/flink/table/expressions/rules/ResolveCallByArgumentsRule.java +++ b/flink-table/flink-table-planner/src/main/java/org/apache/flink/table/expressions/rules/ResolveCallByArgumentsRule.java @@ -31,6 +31,7 @@ import org.apache.flink.table.validate.ValidationFailure; import org.apache.flink.table.validate.ValidationResult; import java.util.List; +import java.util.Optional; import java.util.stream.Collectors; import java.util.stream.IntStream; @@ -87,21 +88,33 @@ final class ResolveCallByArgumentsRule implements ResolverRule { private Expression validateArguments(CallExpression call, PlannerExpression plannerCall) { if (!plannerCall.valid()) { - final String errorMessage; - ValidationResult validationResult = plannerCall.validateInput(); - if (validationResult instanceof ValidationFailure) { - errorMessage = ((ValidationFailure) validationResult).message(); - } else { - errorMessage = String.format("Invalid arguments %s for function: %s", - call.getChildren(), - call.getFunctionDefinition().getName()); - } - throw new ValidationException(errorMessage); + throw new ValidationException( + getValidationErrorMessage(plannerCall) + .orElse("Unexpected behavior, validation failed but can't get error messages!")); } - return call; } + /** + * Return the validation error message of this {@link PlannerExpression} or return the + * validation error message of it's children if it passes the validation. Return empty if + * all validation succeeded. + */ + private Optional<String> getValidationErrorMessage(PlannerExpression plannerCall) { + ValidationResult validationResult = plannerCall.validateInput(); + if (validationResult instanceof ValidationFailure) { + return Optional.of(((ValidationFailure) validationResult).message()); + } else { + for (Expression plannerExpression: plannerCall.getChildren()) { + Optional<String> errorMessage = getValidationErrorMessage((PlannerExpression) plannerExpression); + if (errorMessage.isPresent()) { + return errorMessage; + } + } + } + return Optional.empty(); + } + private Expression castIfNeeded(PlannerExpression childExpression, TypeInformation<?> expectedType) { TypeInformation<?> actualType = childExpression.resultType(); if (actualType.equals(expectedType)) { diff --git a/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/CalcValidationTest.scala b/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/CalcValidationTest.scala index 53ba73c..b9e9f21 100644 --- a/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/CalcValidationTest.scala +++ b/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/CalcValidationTest.scala @@ -148,4 +148,17 @@ class CalcValidationTest extends TableTestBase { "MyTable", 'string) .map("func(string) as a") // do not support TableFunction as input } + + @Test + def testInvalidParameterTypes(): Unit = { + expectedException.expect(classOf[ValidationException]) + expectedException.expectMessage("log('long) fails on input type checking: " + + "[expecting Double on 0th input, get Long].\nOperand should be casted to proper type") + + val util = streamTestUtil() + + util.tableEnv.registerFunction("func", new TableFunc0) + util.addTable[(Int, Long, String)]("MyTable", 'int, 'long, 'string) + .select('int, 'long.log as 'long, 'string) + } } diff --git a/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/GroupWindowValidationTest.scala b/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/GroupWindowValidationTest.scala index 6a11e04..7b7ff87 100644 --- a/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/GroupWindowValidationTest.scala +++ b/flink-table/flink-table-planner/src/test/scala/org/apache/flink/table/api/stream/table/validation/GroupWindowValidationTest.scala @@ -104,7 +104,9 @@ class GroupWindowValidationTest extends TableTestBase { @Test def testTumbleUdAggWithInvalidArgs(): Unit = { expectedException.expect(classOf[ValidationException]) - expectedException.expectMessage("Invalid arguments") + expectedException.expectMessage("Given parameters do not match any signature. \n" + + "Actual: (java.lang.String, java.lang.Integer) \nExpected: (int, int), (long, int), " + + "(long, int, int, java.lang.String)") val util = streamTestUtil() val weightedAvg = new WeightedAvgWithMerge @@ -165,7 +167,9 @@ class GroupWindowValidationTest extends TableTestBase { @Test def testSlideUdAggWithInvalidArgs(): Unit = { expectedException.expect(classOf[ValidationException]) - expectedException.expectMessage("Invalid arguments") + expectedException.expectMessage("Given parameters do not match any signature. \n" + + "Actual: (java.lang.String, java.lang.Integer) \nExpected: (int, int), (long, int), " + + "(long, int, int, java.lang.String)") val util = streamTestUtil() val weightedAvg = new WeightedAvgWithMerge @@ -243,7 +247,9 @@ class GroupWindowValidationTest extends TableTestBase { @Test def testSessionUdAggWithInvalidArgs(): Unit = { expectedException.expect(classOf[ValidationException]) - expectedException.expectMessage("Invalid arguments") + expectedException.expectMessage("Given parameters do not match any signature. \n" + + "Actual: (java.lang.String, java.lang.Integer) \nExpected: (int, int), (long, int), " + + "(long, int, int, java.lang.String)") val util = streamTestUtil() val weightedAvg = new WeightedAvgWithMerge