[
https://issues.apache.org/jira/browse/FLINK-6491?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16004898#comment-16004898
]
ASF GitHub Bot commented on FLINK-6491:
---------------------------------------
Github user fhueske commented on a diff in the pull request:
https://github.com/apache/flink/pull/3863#discussion_r115747425
--- Diff:
flink-libraries/flink-table/src/test/scala/org/apache/flink/table/runtime/harness/HarnessTestBase.scala
---
@@ -19,15 +19,299 @@ package org.apache.flink.table.runtime.harness
import java.util.{Comparator, Queue => JQueue}
+import org.apache.flink.api.common.time.Time
+import org.apache.flink.api.common.typeinfo.BasicTypeInfo.{INT_TYPE_INFO,
LONG_TYPE_INFO, STRING_TYPE_INFO}
import org.apache.flink.api.common.typeinfo.TypeInformation
import org.apache.flink.api.java.functions.KeySelector
+import org.apache.flink.api.java.typeutils.RowTypeInfo
import org.apache.flink.streaming.api.operators.OneInputStreamOperator
import org.apache.flink.streaming.api.watermark.Watermark
import org.apache.flink.streaming.runtime.streamrecord.StreamRecord
import
org.apache.flink.streaming.util.{KeyedOneInputStreamOperatorTestHarness,
TestHarnessUtil}
-import org.apache.flink.table.runtime.types.CRow
+import org.apache.flink.table.api.StreamQueryConfig
+import org.apache.flink.table.codegen.GeneratedAggregationsFunction
+import org.apache.flink.table.functions.AggregateFunction
+import
org.apache.flink.table.functions.aggfunctions.{LongMaxWithRetractAggFunction,
LongMinWithRetractAggFunction, IntSumWithRetractAggFunction}
+import org.apache.flink.table.runtime.aggregate.AggregateUtil
+import org.apache.flink.table.runtime.types.{CRow, CRowTypeInfo}
class HarnessTestBase {
+
+ protected var qConfig =
--- End diff --
I would not make this part of the test base.
IMO, the tests are easier to read if we do not have to check the harness
for all implementations.
> Add QueryConfig to specify state retention time for streaming queries
> ---------------------------------------------------------------------
>
> Key: FLINK-6491
> URL: https://issues.apache.org/jira/browse/FLINK-6491
> Project: Flink
> Issue Type: Bug
> Components: Table API & SQL
> Affects Versions: 1.3.0
> Reporter: Fabian Hueske
> Assignee: sunjincheng
> Priority: Critical
>
> By now we have a couple of streaming operators (group-windows, over-windows,
> non-windowed aggregations) that require operator state. Since state is not
> automatically cleaned-up by Flink, we need to add a mechanism to configure a
> state retention time.
> If configured, a query will retain state for a specified period of state
> inactivity. If state is not accessed within this period of time, it will be
> cleared. I propose to add two parameters for this, a min and a max retention
> time. The min retention time specifies the earliest time and the max
> retention time the latest time when state is cleared. The reasoning for
> having two parameters is that we can avoid to register many timers if we have
> more freedom when to discard state.
> This issue also introduces a QueryConfig object which can be passed to a
> streaming query, when it is emitted to a TableSink or converted to a
> DataStream (append or retraction).
--
This message was sent by Atlassian JIRA
(v6.3.15#6346)