You are viewing a plain text version of this content. The canonical link for it is here.
Posted to issues@flink.apache.org by twalthr <gi...@git.apache.org> on 2016/09/03 07:27:52 UTC

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

GitHub user twalthr opened a pull request:

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

    [FLINK-3580] [table] Add OVERLAPS function

    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.
    
    - [x] 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)
    
    - [x] 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
    
    - [x] 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
    
    This PR implements the SQL OVERLAPS function for Table API and SQL. It allows for checking if two anchored intervals overlap. The SQL documentation is missing as it will be reworked anyway shortly.
    


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

    $ git pull https://github.com/twalthr/flink FLINK-3580_OVERLAPS

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

    https://github.com/apache/flink/pull/2468.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 #2468
    
----
commit edc61d6f85005cad3904a319289669a1ade9c46d
Author: twalthr <tw...@apache.org>
Date:   2016-09-03T06:00:58Z

    [FLINK-3580] [table] Add OVERLAPS function

----


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by fhueske <gi...@git.apache.org>.
Github user fhueske commented on a diff in the pull request:

    https://github.com/apache/flink/pull/2468#discussion_r79826575
  
    --- Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/api/table/expressions/time.scala ---
    @@ -248,3 +249,97 @@ case class LocalTime() extends CurrentTimePoint(SqlTimeTypeInfo.TIME, local = tr
     
     case class LocalTimestamp() extends CurrentTimePoint(SqlTimeTypeInfo.TIMESTAMP, local = true)
     
    +case class TemporalOverlaps(
    +    leftTimePoint: Expression,
    +    leftTemporal: Expression,
    +    rightTimePoint: Expression,
    +    rightTemporal: Expression)
    +  extends Expression {
    +
    +  override private[flink] def children: Seq[Expression] =
    +    Seq(leftTimePoint, leftTemporal, rightTimePoint, rightTemporal)
    +
    +  override private[flink] def resultType: TypeInformation[_] = BOOLEAN_TYPE_INFO
    +
    +  override private[flink] def validateInput(): ExprValidationResult = {
    +    if (!TypeCheckUtils.isTimePoint(leftTimePoint.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTimePoint to be of type " +
    +        s"Time Point, but get ${leftTimePoint.resultType}.")
    +    }
    +    if (!TypeCheckUtils.isTimePoint(rightTimePoint.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires rightTimePoint to be of " +
    +        s"type Time Point, but get ${rightTimePoint.resultType}.")
    +    }
    +    if (leftTimePoint.resultType != rightTimePoint.resultType) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTimePoint and " +
    +        s"rightTimePoint to be of same type.")
    +    }
    +
    +    // leftTemporal is point, then it must be comparable with leftTimePoint
    +    if (TypeCheckUtils.isTimePoint(leftTemporal.resultType)) {
    +      if (leftTemporal.resultType != leftTimePoint.resultType) {
    +        return ValidationFailure(s"TemporalOverlaps operator requires leftTemporal and " +
    +          s"leftTimePoint to be of same type if leftPointOrInterval is of type Time Point.")
    --- End diff --
    
    `leftPointOrInterval` should be `leftTemporal`?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by fhueske <gi...@git.apache.org>.
Github user fhueske commented on a diff in the pull request:

    https://github.com/apache/flink/pull/2468#discussion_r79824726
  
    --- Diff: docs/dev/table_api.md ---
    @@ -1940,6 +1951,17 @@ localTimestamp()
           </td>
         </tr>
     
    +    <tr>
    +      <td>
    +        {% highlight scala %}
    +temporalOverlaps(TIMEPOINT, TEMPORAL, TIMEPOINT, TEMPORAL)
    +{% endhighlight %}
    +      </td>
    +      <td>
    +        <p>Determines whether two anchored time intervals overlap. It evaluates <code>leftTemporal >= rightTimePoint && rightTemporal >= leftTimePoint</code>. E.g. <code>temporalOverlaps('2:55:00'.toTime, 1.hour, '3:30:00'.toTime, 2.hour)</code> leads to true.</p>
    --- End diff --
    
    Rephrase condition


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by fhueske <gi...@git.apache.org>.
Github user fhueske commented on a diff in the pull request:

    https://github.com/apache/flink/pull/2468#discussion_r79827070
  
    --- Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/api/table/expressions/time.scala ---
    @@ -248,3 +249,97 @@ case class LocalTime() extends CurrentTimePoint(SqlTimeTypeInfo.TIME, local = tr
     
     case class LocalTimestamp() extends CurrentTimePoint(SqlTimeTypeInfo.TIMESTAMP, local = true)
     
    +case class TemporalOverlaps(
    +    leftTimePoint: Expression,
    +    leftTemporal: Expression,
    +    rightTimePoint: Expression,
    +    rightTemporal: Expression)
    +  extends Expression {
    +
    +  override private[flink] def children: Seq[Expression] =
    +    Seq(leftTimePoint, leftTemporal, rightTimePoint, rightTemporal)
    +
    +  override private[flink] def resultType: TypeInformation[_] = BOOLEAN_TYPE_INFO
    +
    +  override private[flink] def validateInput(): ExprValidationResult = {
    +    if (!TypeCheckUtils.isTimePoint(leftTimePoint.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTimePoint to be of type " +
    +        s"Time Point, but get ${leftTimePoint.resultType}.")
    +    }
    +    if (!TypeCheckUtils.isTimePoint(rightTimePoint.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires rightTimePoint to be of " +
    +        s"type Time Point, but get ${rightTimePoint.resultType}.")
    +    }
    +    if (leftTimePoint.resultType != rightTimePoint.resultType) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTimePoint and " +
    +        s"rightTimePoint to be of same type.")
    +    }
    +
    +    // leftTemporal is point, then it must be comparable with leftTimePoint
    +    if (TypeCheckUtils.isTimePoint(leftTemporal.resultType)) {
    +      if (leftTemporal.resultType != leftTimePoint.resultType) {
    +        return ValidationFailure(s"TemporalOverlaps operator requires leftTemporal and " +
    +          s"leftTimePoint to be of same type if leftPointOrInterval is of type Time Point.")
    +      }
    +    } else if (!isTimeInterval(leftTemporal.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTemporal to be of " +
    +        s"type Time Point or Time Interval.")
    +    }
    +
    +    // rightTemporal is point, then it must be comparable with rightTimePoint
    +    if (TypeCheckUtils.isTimePoint(rightTemporal.resultType)) {
    +      if (rightTemporal.resultType != rightTimePoint.resultType) {
    +        return ValidationFailure(s"TemporalOverlaps operator requires rightTemporal and " +
    +          s"rightTimePoint to be of same type if rightPointOrInterval is of type Time Point.")
    +      }
    +    } else if (!isTimeInterval(rightTemporal.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires rightTemporal to be of " +
    +        s"type Time Point or Time Interval.")
    +    }
    +    ValidationSuccess
    +  }
    +
    +  override def toString: String = s"temporalOverlaps(${children.mkString(", ")})"
    +
    +  override private[flink] def toRexNode(implicit relBuilder: RelBuilder): RexNode = {
    +    convertOverlaps(
    +      leftTimePoint.toRexNode,
    +      leftTemporal.toRexNode,
    +      rightTimePoint.toRexNode,
    +      rightTemporal.toRexNode,
    +      relBuilder.asInstanceOf[FlinkRelBuilder])
    +  }
    +
    +  /**
    +    * Standard conversion of the OVERLAPS operator.
    +    * Source: [[org.apache.calcite.sql2rel.StandardConvertletTable#convertOverlaps()]]
    +    */
    +  private def convertOverlaps(
    +      leftP: RexNode,
    +      leftT: RexNode,
    +      rightP: RexNode,
    +      rightT: RexNode,
    +      relBuilder: FlinkRelBuilder)
    +    : RexNode = {
    +    // t1 = t0 + t1 if t1 is an interval
    --- End diff --
    
    improve comment. What are `t0`, `t1`, etc.? 


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by fhueske <gi...@git.apache.org>.
Github user fhueske commented on a diff in the pull request:

    https://github.com/apache/flink/pull/2468#discussion_r79824752
  
    --- Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/api/scala/table/expressionDsl.scala ---
    @@ -472,5 +472,30 @@ object localTimestamp {
       }
     }
     
    +/**
    +  * Determines whether two anchored time intervals overlap.
    +  *
    +  * It evaluates: leftTemporal >= rightTimePoint && rightTemporal >= leftTimePoint
    +  *
    +  * e.g. temporalOverlaps("2:55:00".toTime, 1.hour, "3:30:00".toTime, 2.hour) leads to true
    +  */
    +object temporalOverlaps {
    +
    +  /**
    +    * Determines whether two anchored time intervals overlap.
    +    *
    +    * It evaluates: leftTemporal >= rightTimePoint && rightTemporal >= leftTimePoint
    --- End diff --
    
    Rephrase condition


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by asfgit <gi...@git.apache.org>.
Github user asfgit closed the pull request at:

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


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by fhueske <gi...@git.apache.org>.
Github user fhueske commented on a diff in the pull request:

    https://github.com/apache/flink/pull/2468#discussion_r79826729
  
    --- Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/api/table/expressions/time.scala ---
    @@ -248,3 +249,97 @@ case class LocalTime() extends CurrentTimePoint(SqlTimeTypeInfo.TIME, local = tr
     
     case class LocalTimestamp() extends CurrentTimePoint(SqlTimeTypeInfo.TIMESTAMP, local = true)
     
    +case class TemporalOverlaps(
    +    leftTimePoint: Expression,
    +    leftTemporal: Expression,
    +    rightTimePoint: Expression,
    +    rightTemporal: Expression)
    +  extends Expression {
    +
    +  override private[flink] def children: Seq[Expression] =
    +    Seq(leftTimePoint, leftTemporal, rightTimePoint, rightTemporal)
    +
    +  override private[flink] def resultType: TypeInformation[_] = BOOLEAN_TYPE_INFO
    +
    +  override private[flink] def validateInput(): ExprValidationResult = {
    +    if (!TypeCheckUtils.isTimePoint(leftTimePoint.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTimePoint to be of type " +
    +        s"Time Point, but get ${leftTimePoint.resultType}.")
    +    }
    +    if (!TypeCheckUtils.isTimePoint(rightTimePoint.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires rightTimePoint to be of " +
    +        s"type Time Point, but get ${rightTimePoint.resultType}.")
    +    }
    +    if (leftTimePoint.resultType != rightTimePoint.resultType) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTimePoint and " +
    +        s"rightTimePoint to be of same type.")
    +    }
    +
    +    // leftTemporal is point, then it must be comparable with leftTimePoint
    +    if (TypeCheckUtils.isTimePoint(leftTemporal.resultType)) {
    +      if (leftTemporal.resultType != leftTimePoint.resultType) {
    +        return ValidationFailure(s"TemporalOverlaps operator requires leftTemporal and " +
    +          s"leftTimePoint to be of same type if leftPointOrInterval is of type Time Point.")
    +      }
    +    } else if (!isTimeInterval(leftTemporal.resultType)) {
    +      return ValidationFailure(s"TemporalOverlaps operator requires leftTemporal to be of " +
    +        s"type Time Point or Time Interval.")
    +    }
    +
    +    // rightTemporal is point, then it must be comparable with rightTimePoint
    +    if (TypeCheckUtils.isTimePoint(rightTemporal.resultType)) {
    +      if (rightTemporal.resultType != rightTimePoint.resultType) {
    +        return ValidationFailure(s"TemporalOverlaps operator requires rightTemporal and " +
    +          s"rightTimePoint to be of same type if rightPointOrInterval is of type Time Point.")
    --- End diff --
    
    `rightPointOrInterval` should be `rightTemporal`?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by fhueske <gi...@git.apache.org>.
Github user fhueske commented on a diff in the pull request:

    https://github.com/apache/flink/pull/2468#discussion_r79824738
  
    --- Diff: flink-libraries/flink-table/src/main/scala/org/apache/flink/api/scala/table/expressionDsl.scala ---
    @@ -472,5 +472,30 @@ object localTimestamp {
       }
     }
     
    +/**
    +  * Determines whether two anchored time intervals overlap.
    +  *
    +  * It evaluates: leftTemporal >= rightTimePoint && rightTemporal >= leftTimePoint
    --- End diff --
    
    Rephrase condition


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---

[GitHub] flink pull request #2468: [FLINK-3580] [table] Add OVERLAPS function

Posted by fhueske <gi...@git.apache.org>.
Github user fhueske commented on a diff in the pull request:

    https://github.com/apache/flink/pull/2468#discussion_r79824678
  
    --- Diff: docs/dev/table_api.md ---
    @@ -1578,6 +1578,17 @@ localTimestamp()
           </td>
         </tr>
     
    +    <tr>
    +      <td>
    +        {% highlight java %}
    +temporalOverlaps(TIMEPOINT, TEMPORAL, TIMEPOINT, TEMPORAL)
    +{% endhighlight %}
    +      </td>
    +      <td>
    +        <p>Determines whether two anchored time intervals overlap. It evaluates <code>leftTemporal >= rightTimePoint && rightTemporal >= leftTimePoint</code>. E.g. <code>temporalOverlaps("2:55:00".toTime, 1.hour, "3:30:00".toTime, 2.hour)</code> leads to true.</p>
    --- End diff --
    
    I think the condition is hard to understand because it is not clear what a `Temporal` is. How about we say that time point and temporal are transformed into a range defined by two timestamps (start, end) and give the condition as `leftEnd >= rightStart && rightEnd >= leftStart`?


---
If your project is set up for it, you can reply to this email and have your
reply appear on GitHub as well. If your project does not have this feature
enabled and wishes so, or if the feature is enabled but not working, please
contact infrastructure at infrastructure@apache.org or file a JIRA ticket
with INFRA.
---