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 90508a00bb3 [FLINK-35687] JSON_QUERY should return a well formatted nested objects/arrays for ARRAY<STRING> (#24976) 90508a00bb3 is described below commit 90508a00bb310d26f9be245098cda9192647ab14 Author: Dawid Wysakowicz <dwysakow...@apache.org> AuthorDate: Wed Jun 26 08:49:57 2024 +0200 [FLINK-35687] JSON_QUERY should return a well formatted nested objects/arrays for ARRAY<STRING> (#24976) --- .../planner/functions/JsonFunctionsITCase.java | 27 ++++++++++++++++++++++ .../table/runtime/functions/SqlJsonUtils.java | 8 ++++++- 2 files changed, 34 insertions(+), 1 deletion(-) diff --git a/flink-table/flink-table-planner/src/test/java/org/apache/flink/table/planner/functions/JsonFunctionsITCase.java b/flink-table/flink-table-planner/src/test/java/org/apache/flink/table/planner/functions/JsonFunctionsITCase.java index 5108fcdac3d..b8780a34059 100644 --- a/flink-table/flink-table-planner/src/test/java/org/apache/flink/table/planner/functions/JsonFunctionsITCase.java +++ b/flink-table/flink-table-planner/src/test/java/org/apache/flink/table/planner/functions/JsonFunctionsITCase.java @@ -447,6 +447,33 @@ class JsonFunctionsITCase extends BuiltInFunctionTestBase { "JSON_QUERY(f0, '$.a' RETURNING ARRAY<STRING> WITHOUT WRAPPER EMPTY ARRAY ON ERROR)", new String[] {}, DataTypes.ARRAY(DataTypes.STRING())), + + // stringifying RETURNING<ARRAY> + TestSetSpec.forFunction(BuiltInFunctionDefinitions.JSON_QUERY) + .onFieldsWithData( + "{\"items\": [{\"itemId\":1234, \"count\":10}, null, {\"itemId\":4567, \"count\":11}]}", + "{\"items\": [[1234, 2345], null, [\"itemId\", \"count\"]]}", + "{\"arr\": [\"abc\", null, \"def\"]}") + .andDataTypes(STRING(), STRING(), STRING()) + .testResult( + $("f0").jsonQuery("$.items", ARRAY(STRING())), + "JSON_QUERY(f0, '$.items' RETURNING ARRAY<STRING>)", + new String[] { + "{\"itemId\":1234,\"count\":10}", + null, + "{\"itemId\":4567,\"count\":11}" + }, + ARRAY(STRING())) + .testResult( + $("f1").jsonQuery("$.items", ARRAY(STRING())), + "JSON_QUERY(f1, '$.items' RETURNING ARRAY<STRING>)", + new String[] {"[1234,2345]", null, "[\"itemId\",\"count\"]"}, + ARRAY(STRING())) + .testResult( + $("f2").jsonQuery("$.arr", ARRAY(STRING())), + "JSON_QUERY(f2, '$.arr' RETURNING ARRAY<STRING>)", + new String[] {"abc", null, "def"}, + ARRAY(STRING())), TestSetSpec.forFunction(BuiltInFunctionDefinitions.JSON_QUERY) .onFieldsWithData(jsonValue) .andDataTypes(STRING()) diff --git a/flink-table/flink-table-runtime/src/main/java/org/apache/flink/table/runtime/functions/SqlJsonUtils.java b/flink-table/flink-table-runtime/src/main/java/org/apache/flink/table/runtime/functions/SqlJsonUtils.java index 2169f5c470a..09cb0009bf1 100644 --- a/flink-table/flink-table-runtime/src/main/java/org/apache/flink/table/runtime/functions/SqlJsonUtils.java +++ b/flink-table/flink-table-runtime/src/main/java/org/apache/flink/table/runtime/functions/SqlJsonUtils.java @@ -265,7 +265,13 @@ public class SqlJsonUtils { for (int i = 0; i < list.size(); i++) { final Object el = list.get(i); if (el != null) { - arr[i] = StringData.fromString(el.toString()); + final String stringifiedEl; + if (isScalarObject(el)) { + stringifiedEl = String.valueOf(el); + } else { + stringifiedEl = jsonize(el); + } + arr[i] = StringData.fromString(stringifiedEl); } }