Hi all,
Before I raise a Jira issue I wanted to check if this is a known issue - or at
least, whether I'm doing something wrong.
tEnv.executeSql(
"CREATE TABLE primary_2 (" +
" ts TIMESTAMP_LTZ(3)," +
" name STRING," +
" contact ROW<address ROW<line1 STRING, line2 STRING>>," +
" WATERMARK FOR ts AS ts" +
") WITH (" +
" 'connector' = 'datagen'," +
" 'rows-per-second' = '1'," +
" 'scan.parallelism' = '1'," +
" 'fields.name.length' = '1'," +
" 'fields.contact.address.line1.length' = '1'," +
" 'fields.contact.address.line2.length' = '1'" +
")");
tEnv.executeSql("create temporary view primary_2_view as SELECT * FROM
primary_2 MATCH_RECOGNIZE( PARTITION BY name ORDER BY ts MEASURES A.name AS
matchname PATTERN (A) DEFINE A AS name = 'nic');");
tEnv.executeSql("SELECT * FROM primary_2_view").print();
In Flink 1.20.0, this runs correctly (it's not expected to find a pattern). In
Flink 2.20.0 I get a SQL parse exception:
Exception in thread "main" org.apache.flink.table.api.SqlParserException: SQL
parse failed. Encountered "MATCH_RECOGNIZE" at line 2, column 70.
Was expecting one of:
<EOF>
"CROSS" ...
"EXCEPT" ...
"FETCH" ...
"FULL" ...
"GROUP" ...
"HAVING" ...
"INNER" ...
"INTERSECT" ...
"JOIN" ...
"LEFT" ...
"LIMIT" ...
"NATURAL" ...
"OFFSET" ...
"ORDER" ...
"OUTER" ...
"QUALIFY" ...
"RIGHT" ...
"MINUS" ...
"TABLESAMPLE" ...
"UNION" ...
"WHERE" ...
"WINDOW" ...
"(" ...
"," ...
at
org.apache.flink.table.planner.parse.CalciteParser.parse(CalciteParser.java:61)
at
org.apache.flink.table.planner.calcite.FlinkPlannerImpl$ToRelContextImpl.expandView(FlinkPlannerImpl.scala:367)
at
org.apache.calcite.plan.ViewExpanders$1.expandView(ViewExpanders.java:52)
at
org.apache.flink.table.planner.catalog.SqlCatalogViewTable.convertToRel(SqlCatalogViewTable.java:58)
at
org.apache.flink.table.planner.plan.schema.ExpandingPreparingTable.expand(ExpandingPreparingTable.java:70)
at
org.apache.flink.table.planner.plan.schema.ExpandingPreparingTable.toRel(ExpandingPreparingTable.java:57)
at
org.apache.calcite.sql2rel.SqlToRelConverter.toRel(SqlToRelConverter.java:4142)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertIdentifier(SqlToRelConverter.java:2997)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertFrom(SqlToRelConverter.java:2529)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertFrom(SqlToRelConverter.java:2435)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertFrom(SqlToRelConverter.java:2380)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertSelectImpl(SqlToRelConverter.java:758)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertSelect(SqlToRelConverter.java:746)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertQueryRecursive(SqlToRelConverter.java:3967)
at
org.apache.calcite.sql2rel.SqlToRelConverter.convertQuery(SqlToRelConverter.java:650)
at
org.apache.flink.table.planner.calcite.FlinkPlannerImpl.org$apache$flink$table$planner$calcite$FlinkPlannerImpl$$rel(FlinkPlannerImpl.scala:235)
at
org.apache.flink.table.planner.calcite.FlinkPlannerImpl.rel(FlinkPlannerImpl.scala:210)
at
org.apache.flink.table.planner.operations.SqlNodeConvertContext.toRelRoot(SqlNodeConvertContext.java:82)
at
org.apache.flink.table.planner.operations.converters.SqlQueryConverter.convertSqlNode(SqlQueryConverter.java:48)
at
org.apache.flink.table.planner.operations.converters.SqlNodeConverters.convertSqlNode(SqlNodeConverters.java:133)
at
org.apache.flink.table.planner.operations.SqlNodeToOperationConversion.convertValidatedSqlNode(SqlNodeToOperationConversion.java:231)
at
org.apache.flink.table.planner.operations.SqlNodeToOperationConversion.convert(SqlNodeToOperationConversion.java:221)
at
org.apache.flink.table.planner.delegation.ParserImpl.parse(ParserImpl.java:106)
at
org.apache.flink.table.api.internal.TableEnvironmentImpl.executeSql(TableEnvironmentImpl.java:937)
at
com.example.UnionTablesExample.executeUnionQuery(UnionTablesExample.java:54)
at com.example.UnionTablesExample.main(UnionTablesExample.java:23)
Caused by: org.apache.calcite.sql.parser.SqlParseException: Encountered
"MATCH_RECOGNIZE" at line 2, column 70.
Was expecting one of:
<EOF>
"CROSS" ...
"EXCEPT" ...
"FETCH" ...
"FULL" ...
"GROUP" ...
"HAVING" ...
"INNER" ...
"INTERSECT" ...
"JOIN" ...
"LEFT" ...
"LIMIT" ...
"NATURAL" ...
"OFFSET" ...
"ORDER" ...
"OUTER" ...
"QUALIFY" ...
"RIGHT" ...
"MINUS" ...
"TABLESAMPLE" ...
"UNION" ...
"WHERE" ...
"WINDOW" ...
"(" ...
"," ...
at
org.apache.flink.sql.parser.impl.FlinkSqlParserImpl.convertException(FlinkSqlParserImpl.java:558)
at
org.apache.flink.sql.parser.impl.FlinkSqlParserImpl.normalizeException(FlinkSqlParserImpl.java:304)
at
org.apache.calcite.sql.parser.SqlParser.handleException(SqlParser.java:159)
at
org.apache.calcite.sql.parser.SqlParser.parseQuery(SqlParser.java:174)
at org.apache.calcite.sql.parser.SqlParser.parseStmt(SqlParser.java:199)
at
org.apache.flink.table.planner.parse.CalciteParser.parse(CalciteParser.java:56)
... 25 more
Caused by: org.apache.flink.sql.parser.impl.ParseException: Encountered
"MATCH_RECOGNIZE" at line 2, column 70.
Was expecting one of:
<EOF>
"CROSS" ...
"EXCEPT" ...
"FETCH" ...
"FULL" ...
"GROUP" ...
"HAVING" ...
"INNER" ...
"INTERSECT" ...
"JOIN" ...
"LEFT" ...
"LIMIT" ...
"NATURAL" ...
"OFFSET" ...
"ORDER" ...
"OUTER" ...
"QUALIFY" ...
"RIGHT" ...
"MINUS" ...
"TABLESAMPLE" ...
"UNION" ...
"WHERE" ...
"WINDOW" ...
"(" ...
"," ...
at
org.apache.flink.sql.parser.impl.FlinkSqlParserImpl.generateParseException(FlinkSqlParserImpl.java:54079)
at
org.apache.flink.sql.parser.impl.FlinkSqlParserImpl.jj_consume_token(FlinkSqlParserImpl.java:53884)
at
org.apache.flink.sql.parser.impl.FlinkSqlParserImpl.SqlStmtEof(FlinkSqlParserImpl.java:4483)
at
org.apache.flink.sql.parser.impl.FlinkSqlParserImpl.parseSqlStmtEof(FlinkSqlParserImpl.java:352)
at
org.apache.calcite.sql.parser.SqlParser.parseQuery(SqlParser.java:172)
... 27 more
I've read through the calcite 1.35 release notes and I can't find anything
obvious regarding a change to MATCH_RECOGNIZE.
Also - if I don't use a temporary view, and instead just execute the
MATCH_RECOGNIZE select statement, it works correctly. So it seems related to
the use of a view. But I'd like to use the output in a downstream view before
sending the result to a sink.
--
Nic Townsend
IBM Event Processing
Architect / Senior Engineer
Slack: @nictownsend
Bluesky: @nict0wnsend.bsky.social
Unless otherwise stated above:
IBM United Kingdom Limited
Registered in England and Wales with number 741598
Registered office: Building C, IBM Hursley Office, Hursley Park Road,
Winchester, Hampshire SO21 2JN