Uploaded image for project: 'Flink'
  1. Flink
  2. FLINK-6930

Selecting window start / end on row-based Tumble/Slide window causes NPE

    Details

    • Type: Bug
    • Status: Closed
    • Priority: Major
    • Resolution: Fixed
    • Affects Version/s: 1.3.0, 1.4.0
    • Fix Version/s: 1.3.1, 1.4.0
    • Component/s: Table API & SQL
    • Labels:
      None

      Description

      Selecting the start and end properties of a row-based window causes a NullPointerException.
      The following program:

      val windowedTable = table
            .window(Tumble over 2.rows on 'proctime as 'w)
            .groupBy('w, 'string)
            .select('string as 'n, 'int.count as 'cnt, 'w.start as 's, 'w.end as 'e)
      

      causes

      Caused by: java.lang.NullPointerException
      	at org.apache.calcite.runtime.SqlFunctions.toLong(SqlFunctions.java:1556)
      	at org.apache.calcite.runtime.SqlFunctions.toLong(SqlFunctions.java:1551)
      	at DataStreamCalcRule$40.processElement(Unknown Source)
      	at org.apache.flink.table.runtime.CRowProcessRunner.processElement(CRowProcessRunner.scala:67)
      	at org.apache.flink.table.runtime.CRowProcessRunner.processElement(CRowProcessRunner.scala:35)
      	at org.apache.flink.streaming.api.operators.ProcessOperator.processElement(ProcessOperator.java:66)
      	at org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.pushToOperator(OperatorChain.java:528)
      	at org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:503)
      	at org.apache.flink.streaming.runtime.tasks.OperatorChain$CopyingChainingOutput.collect(OperatorChain.java:483)
      	at org.apache.flink.streaming.api.operators.AbstractStreamOperator$CountingOutput.collect(AbstractStreamOperator.java:890)
      	at org.apache.flink.streaming.api.operators.AbstractStreamOperator$CountingOutput.collect(AbstractStreamOperator.java:868)
      	at org.apache.flink.streaming.api.operators.TimestampedCollector.collect(TimestampedCollector.java:51)
      	at org.apache.flink.table.runtime.aggregate.IncrementalAggregateWindowFunction.apply(IncrementalAggregateWindowFunction.scala:75)
      	at org.apache.flink.table.runtime.aggregate.IncrementalAggregateWindowFunction.apply(IncrementalAggregateWindowFunction.scala:37)
      	at org.apache.flink.streaming.runtime.operators.windowing.functions.InternalSingleValueWindowFunction.process(InternalSingleValueWindowFunction.java:46)
      	at org.apache.flink.streaming.runtime.operators.windowing.WindowOperator.emitWindowContents(WindowOperator.java:599)
      	at org.apache.flink.streaming.runtime.operators.windowing.WindowOperator.processElement(WindowOperator.java:456)
      	at org.apache.flink.streaming.runtime.io.StreamInputProcessor.processInput(StreamInputProcessor.java:207)
      	at org.apache.flink.streaming.runtime.tasks.OneInputStreamTask.run(OneInputStreamTask.java:69)
      	at org.apache.flink.streaming.runtime.tasks.StreamTask.invoke(StreamTask.java:265)
      	at org.apache.flink.runtime.taskmanager.Task.run(Task.java:702)
      	at java.lang.Thread.run(Thread.java:745)
      

      We should validate that the start and end window properties are not accessed if the window is defined on row-counts.

        Issue Links

          Activity

          Hide
          githubbot ASF GitHub Bot added a comment -

          GitHub user wuchong opened a pull request:

          https://github.com/apache/flink/pull/4133

          FLINK-6930 [table] Forbid selecting window start/end on row-based Tumble/Slide windows

          Thanks for contributing to Apache Flink. Before you open your pull request, please take the following check list into consideration.
          If your changes take all of the items into account, feel free to open your pull request. For more information and/or questions please refer to the [How To Contribute guide](http://flink.apache.org/how-to-contribute.html).
          In addition to going through the list, please provide a meaningful description of your changes.

          • [ ] General
          • The pull request references the related JIRA issue ("[FLINK-XXX] Jira title text")
          • The pull request addresses only one issue
          • Each commit in the PR has a meaningful commit message (including the JIRA id)
          • [ ] Documentation
          • Documentation has been added for new functionality
          • Old documentation affected by the pull request has been updated
          • JavaDoc for public methods has been added
          • [ ] Tests & Build
          • Functionality added by the pull request is covered by tests
          • `mvn clean verify` has been executed successfully locally or a Travis build has passed

          You can merge this pull request into a Git repository by running:

          $ git pull https://github.com/wuchong/flink forbid-rowbased-window-start

          Alternatively you can review and apply these changes as the patch at:

          https://github.com/apache/flink/pull/4133.patch

          To close this pull request, make a commit to your master/trunk branch
          with (at least) the following in the commit message:

          This closes #4133


          commit 4cf422acd25b18ebc285ace542d49ebc646707db
          Author: Jark Wu <wuchong.wc@alibaba-inc.com>
          Date: 2017-06-16T03:49:31Z

          FLINK-6930 [table] Forbid selecting window start/end on row-based Tumble/Slide windows


          Show
          githubbot ASF GitHub Bot added a comment - GitHub user wuchong opened a pull request: https://github.com/apache/flink/pull/4133 FLINK-6930 [table] Forbid selecting window start/end on row-based Tumble/Slide windows Thanks for contributing to Apache Flink. Before you open your pull request, please take the following check list into consideration. If your changes take all of the items into account, feel free to open your pull request. For more information and/or questions please refer to the [How To Contribute guide] ( http://flink.apache.org/how-to-contribute.html ). In addition to going through the list, please provide a meaningful description of your changes. [ ] General The pull request references the related JIRA issue (" [FLINK-XXX] Jira title text") The pull request addresses only one issue Each commit in the PR has a meaningful commit message (including the JIRA id) [ ] Documentation Documentation has been added for new functionality Old documentation affected by the pull request has been updated JavaDoc for public methods has been added [ ] Tests & Build Functionality added by the pull request is covered by tests `mvn clean verify` has been executed successfully locally or a Travis build has passed You can merge this pull request into a Git repository by running: $ git pull https://github.com/wuchong/flink forbid-rowbased-window-start Alternatively you can review and apply these changes as the patch at: https://github.com/apache/flink/pull/4133.patch To close this pull request, make a commit to your master/trunk branch with (at least) the following in the commit message: This closes #4133 commit 4cf422acd25b18ebc285ace542d49ebc646707db Author: Jark Wu <wuchong.wc@alibaba-inc.com> Date: 2017-06-16T03:49:31Z FLINK-6930 [table] Forbid selecting window start/end on row-based Tumble/Slide windows
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user hustfxj commented on the issue:

          https://github.com/apache/flink/pull/4133

          +1

          Show
          githubbot ASF GitHub Bot added a comment - Github user hustfxj commented on the issue: https://github.com/apache/flink/pull/4133 +1
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user sunjincheng121 commented on a diff in the pull request:

          https://github.com/apache/flink/pull/4133#discussion_r122378782

          — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala —
          @@ -652,6 +653,21 @@ case class WindowAggregate(
          case ValidationSuccess => // ok
          }

          + // validate property
          + if (propertyExpressions.nonEmpty) {
          + resolvedWindowAggregate.window match {
          + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) =>
          + failValidation("Window start and Window end are not accessed " +
          + "on row-counts Tumbling window.")
          +
          + case SlidingGroupWindow(_, _, size, _) if isRowCountLiteral(size) =>
          + failValidation("Window start and Window end are not accessed " +
          + "on row-counts Sliding window.")
          — End diff –

          `row-counts` -> `row-count`

          Show
          githubbot ASF GitHub Bot added a comment - Github user sunjincheng121 commented on a diff in the pull request: https://github.com/apache/flink/pull/4133#discussion_r122378782 — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala — @@ -652,6 +653,21 @@ case class WindowAggregate( case ValidationSuccess => // ok } + // validate property + if (propertyExpressions.nonEmpty) { + resolvedWindowAggregate.window match { + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) => + failValidation("Window start and Window end are not accessed " + + "on row-counts Tumbling window.") + + case SlidingGroupWindow(_, _, size, _) if isRowCountLiteral(size) => + failValidation("Window start and Window end are not accessed " + + "on row-counts Sliding window.") — End diff – `row-counts` -> `row-count`
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user sunjincheng121 commented on a diff in the pull request:

          https://github.com/apache/flink/pull/4133#discussion_r122378757

          — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala —
          @@ -652,6 +653,21 @@ case class WindowAggregate(
          case ValidationSuccess => // ok
          }

          + // validate property
          + if (propertyExpressions.nonEmpty) {
          + resolvedWindowAggregate.window match {
          + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) =>
          + failValidation("Window start and Window end are not accessed " +
          + "on row-counts Tumbling window.")
          — End diff –

          `row-counts` -> `row-count`

          Show
          githubbot ASF GitHub Bot added a comment - Github user sunjincheng121 commented on a diff in the pull request: https://github.com/apache/flink/pull/4133#discussion_r122378757 — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala — @@ -652,6 +653,21 @@ case class WindowAggregate( case ValidationSuccess => // ok } + // validate property + if (propertyExpressions.nonEmpty) { + resolvedWindowAggregate.window match { + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) => + failValidation("Window start and Window end are not accessed " + + "on row-counts Tumbling window.") — End diff – `row-counts` -> `row-count`
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user fhueske commented on a diff in the pull request:

          https://github.com/apache/flink/pull/4133#discussion_r122388804

          — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala —
          @@ -652,6 +653,21 @@ case class WindowAggregate(
          case ValidationSuccess => // ok
          }

          + // validate property
          + if (propertyExpressions.nonEmpty) {
          + resolvedWindowAggregate.window match {
          + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) =>
          + failValidation("Window start and Window end are not accessed " +
          + "on row-counts Tumbling window.")
          — End diff –

          How about `"Window start and Window end cannot be selected for a row-count Tumbing window."`?

          Show
          githubbot ASF GitHub Bot added a comment - Github user fhueske commented on a diff in the pull request: https://github.com/apache/flink/pull/4133#discussion_r122388804 — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala — @@ -652,6 +653,21 @@ case class WindowAggregate( case ValidationSuccess => // ok } + // validate property + if (propertyExpressions.nonEmpty) { + resolvedWindowAggregate.window match { + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) => + failValidation("Window start and Window end are not accessed " + + "on row-counts Tumbling window.") — End diff – How about `"Window start and Window end cannot be selected for a row-count Tumbing window."`?
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user fhueske commented on the issue:

          https://github.com/apache/flink/pull/4133

          Thanks for the fix @wuchong!
          I would adjust the error message as proposed but otherwise it's good to merge.

          Thanks, Fabian

          Show
          githubbot ASF GitHub Bot added a comment - Github user fhueske commented on the issue: https://github.com/apache/flink/pull/4133 Thanks for the fix @wuchong! I would adjust the error message as proposed but otherwise it's good to merge. Thanks, Fabian
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user wuchong commented on a diff in the pull request:

          https://github.com/apache/flink/pull/4133#discussion_r122389818

          — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala —
          @@ -652,6 +653,21 @@ case class WindowAggregate(
          case ValidationSuccess => // ok
          }

          + // validate property
          + if (propertyExpressions.nonEmpty) {
          + resolvedWindowAggregate.window match {
          + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) =>
          + failValidation("Window start and Window end are not accessed " +
          + "on row-counts Tumbling window.")
          — End diff –

          Thank you @sunjincheng121 @fhueske for the suggestion.

          Show
          githubbot ASF GitHub Bot added a comment - Github user wuchong commented on a diff in the pull request: https://github.com/apache/flink/pull/4133#discussion_r122389818 — Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/table/plan/logical/operators.scala — @@ -652,6 +653,21 @@ case class WindowAggregate( case ValidationSuccess => // ok } + // validate property + if (propertyExpressions.nonEmpty) { + resolvedWindowAggregate.window match { + case TumblingGroupWindow(_, _, size) if isRowCountLiteral(size) => + failValidation("Window start and Window end are not accessed " + + "on row-counts Tumbling window.") — End diff – Thank you @sunjincheng121 @fhueske for the suggestion.
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user wuchong commented on the issue:

          https://github.com/apache/flink/pull/4133

          Updated

          Show
          githubbot ASF GitHub Bot added a comment - Github user wuchong commented on the issue: https://github.com/apache/flink/pull/4133 Updated
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user fhueske commented on the issue:

          https://github.com/apache/flink/pull/4133

          +1 to merge

          Show
          githubbot ASF GitHub Bot added a comment - Github user fhueske commented on the issue: https://github.com/apache/flink/pull/4133 +1 to merge
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user sunjincheng121 commented on the issue:

          https://github.com/apache/flink/pull/4133

          +1 to merge

          Show
          githubbot ASF GitHub Bot added a comment - Github user sunjincheng121 commented on the issue: https://github.com/apache/flink/pull/4133 +1 to merge
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user wuchong commented on the issue:

          https://github.com/apache/flink/pull/4133

          merging...

          Show
          githubbot ASF GitHub Bot added a comment - Github user wuchong commented on the issue: https://github.com/apache/flink/pull/4133 merging...
          Hide
          jark Jark Wu added a comment -

          Fixed in 06e63386e56e0bf73030b1bf31890c8103592fc0

          Show
          jark Jark Wu added a comment - Fixed in 06e63386e56e0bf73030b1bf31890c8103592fc0
          Hide
          githubbot ASF GitHub Bot added a comment -

          Github user asfgit closed the pull request at:

          https://github.com/apache/flink/pull/4133

          Show
          githubbot ASF GitHub Bot added a comment - Github user asfgit closed the pull request at: https://github.com/apache/flink/pull/4133
          Hide
          fhueske Fabian Hueske added a comment -

          Hi Jark Wu, can you cherry-pick the fix to the 1.3-release branch as well? Thank you

          Show
          fhueske Fabian Hueske added a comment - Hi Jark Wu , can you cherry-pick the fix to the 1.3-release branch as well? Thank you
          Hide
          fhueske Fabian Hueske added a comment -

          I'll merge it to 1.3 as well to get the fix in the next RC for 1.3.1

          Show
          fhueske Fabian Hueske added a comment - I'll merge it to 1.3 as well to get the fix in the next RC for 1.3.1
          Hide
          fhueske Fabian Hueske added a comment -

          Fixed for 1.3.1 with 2321898943c223241794c6a4f387430633c954fb

          Show
          fhueske Fabian Hueske added a comment - Fixed for 1.3.1 with 2321898943c223241794c6a4f387430633c954fb

            People

            • Assignee:
              jark Jark Wu
              Reporter:
              fhueske Fabian Hueske
            • Votes:
              0 Vote for this issue
              Watchers:
              4 Start watching this issue

              Dates

              • Created:
                Updated:
                Resolved:

                Development