[ 
https://issues.apache.org/jira/browse/FLINK-5047?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=15898181#comment-15898181
 ] 

ASF GitHub Bot commented on FLINK-5047:
---------------------------------------

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

    https://github.com/apache/flink/pull/3364#discussion_r104523761
  
    --- Diff: 
flink-libraries/flink-table/src/main/scala/org/apache/flink/table/runtime/aggregate/DataSetSlideTimeWindowAggReduceCombineFunction.scala
 ---
    @@ -0,0 +1,84 @@
    +/*
    + * Licensed to the Apache Software Foundation (ASF) under one
    + * or more contributor license agreements.  See the NOTICE file
    + * distributed with this work for additional information
    + * regarding copyright ownership.  The ASF licenses this file
    + * to you under the Apache License, Version 2.0 (the
    + * "License"); you may not use this file except in compliance
    + * with the License.  You may obtain a copy of the License at
    + *
    + *     http://www.apache.org/licenses/LICENSE-2.0
    + *
    + * Unless required by applicable law or agreed to in writing, software
    + * distributed under the License is distributed on an "AS IS" BASIS,
    + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
    + * See the License for the specific language governing permissions and
    + * limitations under the License.
    + */
    +package org.apache.flink.table.runtime.aggregate
    +
    +import java.lang.Iterable
    +
    +import org.apache.flink.api.common.functions.CombineFunction
    +import org.apache.flink.api.common.typeinfo.TypeInformation
    +import org.apache.flink.types.Row
    +
    +/**
    +  * Wraps the aggregate logic inside of
    +  * [[org.apache.flink.api.java.operators.GroupReduceOperator]] and
    +  * [[org.apache.flink.api.java.operators.GroupCombineOperator]].
    +  *
    +  * It is used for sliding on batch for time-windows.
    +  *
    +  * @param aggregates aggregate functions
    +  * @param groupingKeysLength number of grouping keys
    +  * @param timeFieldPos position of aligned time field
    +  * @param windowSize window size of the sliding window
    +  * @param windowSlide window slide of the sliding window
    +  * @param returnType return type of this function
    +  */
    +class DataSetSlideTimeWindowAggReduceCombineFunction(
    +    aggregates: Array[Aggregate[_]],
    +    groupingKeysLength: Int,
    +    timeFieldPos: Int,
    +    windowSize: Long,
    +    windowSlide: Long,
    +    returnType: TypeInformation[Row])
    +  extends DataSetSlideTimeWindowAggReduceGroupFunction(
    +    aggregates,
    +    groupingKeysLength,
    +    timeFieldPos,
    +    windowSize,
    +    windowSlide,
    +    returnType)
    +  with CombineFunction[Row, Row] {
    +
    +  override def combine(records: Iterable[Row]): Row = {
    +    // initiate intermediate aggregate value
    +    aggregates.foreach(_.initiate(aggregateBuffer))
    +
    +    val iterator = records.iterator()
    +    while (iterator.hasNext) {
    +      val record = iterator.next()
    +
    +      // merge intermediate aggregate value to buffer
    +      aggregates.foreach(_.merge(record, aggregateBuffer))
    +
    +      // check if this record is the last record
    +      if (!iterator.hasNext) {
    +
    +        // set group keys value to buffer
    +        for (i <- 0 until groupingKeysLength) {
    +          aggregateBuffer.setField(i, record.getField(i))
    +        }
    +
    +        aggregateBuffer.setField(timeFieldPos, 
record.getField(timeFieldPos))
    +
    +        return aggregateBuffer
    +      }
    +    }
    +
    +    // this code path should never be reached as we return before the loop 
finishes
    +    throw new IllegalArgumentException("Group is empty. This should never 
happen.")
    --- End diff --
    
    can be removed. It's the responsibility of the DataSet API to call the user 
functions correctly. And even if it would not, this function would behave 
correctly.


> Add sliding group-windows for batch tables
> ------------------------------------------
>
>                 Key: FLINK-5047
>                 URL: https://issues.apache.org/jira/browse/FLINK-5047
>             Project: Flink
>          Issue Type: Sub-task
>          Components: Table API & SQL
>            Reporter: Jark Wu
>            Assignee: Timo Walther
>
> Add Slide group-windows for batch tables as described in 
> [FLIP-11|https://cwiki.apache.org/confluence/display/FLINK/FLIP-11%3A+Table+API+Stream+Aggregations].
> There are two ways to implement sliding windows for batch:
> 1. replicate the output in order to assign keys for overlapping windows. This 
> is probably the more straight-forward implementation and supports any 
> aggregation function but blows up the data volume.
> 2. if the aggregation functions are combinable / pre-aggregatable, we can 
> also find the largest tumbling window size from which the sliding windows can 
> be assembled. This is basically the technique used to express sliding windows 
> with plain SQL (GROUP BY + OVER clauses). For a sliding window Slide(10 
> minutes, 2 minutes) this would mean to first compute aggregates of 
> non-overlapping (tumbling) 2 minute windows and assembling consecutively 5 of 
> these into a sliding window (could be done in a MapPartition with sorted 
> input). The implementation could be done as an optimizer rule to split the 
> sliding aggregate into a tumbling aggregate and a SQL WINDOW operator. Maybe 
> it makes sense to implement the WINDOW clause first and reuse this for 
> sliding windows.
> 3. There is also a third, hybrid solution: Doing the pre-aggregation on the 
> largest non-overlapping windows (as in 2) and replicating these results and 
> processing those as in the 1) approach. The benefits of this is that it a) is 
> based on the implementation that supports non-combinable aggregates (which is 
> required in any case) and b) that it does not require the implementation of 
> the SQL WINDOW operator. Internally, this can be implemented again as an 
> optimizer rule that translates the SlidingWindow into a pre-aggregating 
> TublingWindow and a final SlidingWindow (with replication).
> see FLINK-4692 for more discussion



--
This message was sent by Atlassian JIRA
(v6.3.15#6346)

Reply via email to