Table API

The Table API is a unified, relational API for stream and batch processing. Table API queries can be run on batch or streaming input without modifications. The Table API is a super set of the SQL language and is specially designed for working with Apache Flink. The Table API is a language-integrated API for Scala, Java and Python. Instead of specifying queries as String values as common with SQL, Table API queries are defined in a language-embedded style in Java, Scala or Python with IDE support like autocompletion and syntax validation.

The Table API shares many concepts and parts of its API with Flink’s SQL integration. Have a look at the Common Concepts & API to learn how to register tables or to create a Table object. The Streaming Concepts pages discuss streaming specific concepts such as dynamic tables and time attributes.

The following examples assume a registered table called Orders with attributes (a, b, c, rowtime). The rowtime field is either a logical time attribute in streaming or a regular timestamp field in batch.

Overview & Examples

The Table API is available for Scala, Java and Python. The Scala Table API leverages on Scala expressions, the Java Table API supports both Expression DSL and strings which are parsed and converted into equivalent expressions, the Python Table API currently only supports strings which are parsed and converted into equivalent expressions.

The following example shows the differences between the Scala, Java and Python Table API. The table program is executed in a batch environment. It scans the Orders table, groups by field a, and counts the resulting rows per group.

Java

The Java Table API is enabled by importing org.apache.flink.table.api.java.*. The following example shows how a Java Table API program is constructed and how expressions are specified as strings. For the Expression DSL it is also necessary to import static org.apache.flink.table.api.Expressions.*

  1. import org.apache.flink.table.api.*;
  2. import static org.apache.flink.table.api.Expressions.*;
  3. EnvironmentSettings settings = EnvironmentSettings
  4. .newInstance()
  5. .inStreamingMode()
  6. .build();
  7. TableEnvironment tEnv = TableEnvironment.create(env);
  8. // register Orders table in table environment
  9. // ...
  10. // specify table program
  11. Table orders = tEnv.from("Orders"); // schema (a, b, c, rowtime)
  12. Table counts = orders
  13. .groupBy($("a"))
  14. .select($("a"), $("b").count().as("cnt"));
  15. // conversion to DataSet
  16. DataSet<Row> result = tEnv.toDataSet(counts, Row.class);
  17. result.print();

Scala

The Scala Table API is enabled by importing org.apache.flink.table.api._, org.apache.flink.api.scala._, and org.apache.flink.table.api.bridge.scala._ (for bridging to/from DataStream).

The following example shows how a Scala Table API program is constructed. Table fields are referenced using Scala’s String interpolation using a dollar character ($).

  1. import org.apache.flink.api.scala._
  2. import org.apache.flink.table.api._
  3. import org.apache.flink.table.api.bridge.scala._
  4. // environment configuration
  5. val settings = EnvironmentSettings
  6. .newInstance()
  7. .inStreamingMode()
  8. .build();
  9. val tEnv = TableEnvironment.create(settings);
  10. // register Orders table in table environment
  11. // ...
  12. // specify table program
  13. val orders = tEnv.from("Orders") // schema (a, b, c, rowtime)
  14. val result = orders
  15. .groupBy($"a")
  16. .select($"a", $"b".count as "cnt")
  17. .toDataSet[Row] // conversion to DataSet
  18. .print()

Python

The following example shows how a Python Table API program is constructed and how expressions are specified as strings.

  1. from pyflink.table import *
  2. # environment configuration
  3. t_env = BatchTableEnvironment.create(
  4. environment_settings=EnvironmentSettings.new_instance().in_batch_mode().use_blink_planner().build())
  5. # register Orders table and Result table sink in table environment
  6. source_data_path = "/path/to/source/directory/"
  7. result_data_path = "/path/to/result/directory/"
  8. source_ddl = f"""
  9. create table Orders(
  10. a VARCHAR,
  11. b BIGINT,
  12. c BIGINT,
  13. rowtime TIMESTAMP(3),
  14. WATERMARK FOR rowtime AS rowtime - INTERVAL '1' SECOND
  15. ) with (
  16. 'connector' = 'filesystem',
  17. 'format' = 'csv',
  18. 'path' = '{source_data_path}'
  19. )
  20. """
  21. t_env.execute_sql(source_ddl)
  22. sink_ddl = f"""
  23. create table `Result`(
  24. a VARCHAR,
  25. cnt BIGINT
  26. ) with (
  27. 'connector' = 'filesystem',
  28. 'format' = 'csv',
  29. 'path' = '{result_data_path}'
  30. )
  31. """
  32. t_env.execute_sql(sink_ddl)
  33. # specify table program
  34. orders = t_env.from_path("Orders") # schema (a, b, c, rowtime)
  35. orders.group_by("a").select(orders.a, orders.b.count.alias('cnt')).execute_insert("result").wait()

The next example shows a more complex Table API program. The program scans again the Orders table. It filters null values, normalizes the field a of type String, and calculates for each hour and product a the average billing amount b.

Java

  1. // environment configuration
  2. // ...
  3. // specify table program
  4. Table orders = tEnv.from("Orders"); // schema (a, b, c, rowtime)
  5. Table result = orders
  6. .filter(
  7. and(
  8. $("a").isNotNull(),
  9. $("b").isNotNull(),
  10. $("c").isNotNull()
  11. ))
  12. .select($("a").lowerCase().as("a"), $("b"), $("rowtime"))
  13. .window(Tumble.over(lit(1).hours()).on($("rowtime")).as("hourlyWindow"))
  14. .groupBy($("hourlyWindow"), $("a"))
  15. .select($("a"), $("hourlyWindow").end().as("hour"), $("b").avg().as("avgBillingAmount"));

Scala

  1. // environment configuration
  2. // ...
  3. // specify table program
  4. val orders: Table = tEnv.from("Orders") // schema (a, b, c, rowtime)
  5. val result: Table = orders
  6. .filter($"a".isNotNull && $"b".isNotNull && $"c".isNotNull)
  7. .select($"a".lowerCase() as "a", $"b", $"rowtime")
  8. .window(Tumble over 1.hour on $"rowtime" as "hourlyWindow")
  9. .groupBy($"hourlyWindow", $"a")
  10. .select($"a", $"hourlyWindow".end as "hour", $"b".avg as "avgBillingAmount")

Python

  1. # specify table program
  2. from pyflink.table.expressions import col, lit
  3. orders = t_env.from_path("Orders") # schema (a, b, c, rowtime)
  4. result = orders.filter(orders.a.is_not_null & orders.b.is_not_null & orders.c.is_not_null) \
  5. .select(orders.a.lower_case.alias('a'), orders.b, orders.rowtime) \
  6. .window(Tumble.over(lit(1).hour).on(orders.rowtime).alias("hourly_window")) \
  7. .group_by(col('hourly_window'), col('a')) \
  8. .select(col('a'), col('hourly_window').end.alias('hour'), b.avg.alias('avg_billing_amount'))

Since the Table API is a unified API for batch and streaming data, both example programs can be executed on batch and streaming inputs without any modification of the table program itself. In both cases, the program produces the same results given that streaming records are not late (see Streaming Concepts for details).

Operations

The Table API supports the following operations. Please note that not all operations are available in both batch and streaming yet; they are tagged accordingly.

Scan, Projection, and Filter

From

Batch Streaming

Similar to the FROM clause in a SQL query. Performs a scane of registered table.

Java

  1. Table orders = tableEnv.from("Orders");

Scala

  1. val orders = tableEnv.from("Orders")

Python

  1. orders = t_env.from_path("Orders")

FromValues

Batch Streaming

Similar to the VALUES clause in a SQL query. Produces an inline table out of the provided rows.

You can use a row(...) expression to create composite rows:

Java

  1. Table table = tEnv.fromValues(
  2. row(1, "ABC"),
  3. row(2L, "ABCDE")
  4. );

Scala

  1. table = tEnv.fromValues(
  2. row(1, "ABC"),
  3. row(2L, "ABCDE")
  4. )

Python

  1. table = t_env.from_elements([(1, 'ABC'), (2, 'ABCDE')])

will produce a Table with a schema as follows:

  1. root
  2. |-- f0: BIGINT NOT NULL // original types INT and BIGINT are generalized to BIGINT
  3. |-- f1: VARCHAR(5) NOT NULL // original types CHAR(3) and CHAR(5) are generalized
  4. // to VARCHAR(5). VARCHAR is used instead of CHAR so that
  5. // no padding is applied

The method will derive the types automatically from the input expressions. If types at a certain position differ, the method will try to find a common super type for all types. If a common super type does not exist, an exception will be thrown.

You can also specify the requested type explicitly. It might be helpful for assigning more generic types like e.g. DECIMAL or naming the columns.

Java

  1. Table table = tEnv.fromValues(
  2. DataTypes.ROW(
  3. DataTypes.FIELD("id", DataTypes.DECIMAL(10, 2)),
  4. DataTypes.FIELD("name", DataTypes.STRING())
  5. ),
  6. row(1, "ABC"),
  7. row(2L, "ABCDE")
  8. );

Scala

  1. val table = tEnv.fromValues(
  2. DataTypes.ROW(
  3. DataTypes.FIELD("id", DataTypes.DECIMAL(10, 2)),
  4. DataTypes.FIELD("name", DataTypes.STRING())
  5. ),
  6. row(1, "ABC"),
  7. row(2L, "ABCDE")
  8. )

Python

  1. table = t_env.from_elements(
  2. [(1, 'ABC'), (2, 'ABCDE')],
  3. schema=DataTypes.Row([DataTypes.FIELD('id', DataTypes.DECIMAL(10, 2)),
  4. DataTypes.FIELD('name', DataTypes.STRING())]))

will produce a Table with the following schema:

  1. root
  2. |-- id: DECIMAL(10, 2)
  3. |-- name: STRING

Select

Batch Streaming

Similar to a SQL SELECT statement. Performs a select operation.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.select($("a"), $("c").as("d"));

Scala

  1. val orders = tableEnv.from("Orders")
  2. Table result = orders.select($"a", $"c" as "d");

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.select(orders.a, orders.c.alias('d'))

You can use star (*) to act as a wild card, selecting all of the columns in the table.

Java

  1. Table result = orders.select($("*"));

Scala

  1. Table result = orders.select($"*")

Python

  1. from pyflink.table.expressions import col
  2. result = orders.select(col("*"))

As

Batch Streaming

Renames fields.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.as("x, y, z, t");

scala

  1. val orders: Table = tableEnv.from("Orders").as("x", "y", "z", "t")

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.alias("x, y, z, t")

Where / Filter

Batch Streaming

Similar to a SQL WHERE clause. Filters out rows that do not pass the filter predicate.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.where($("b").isEqual("red"));

Scala

  1. val orders: Table = tableEnv.from("Orders")
  2. val result = orders.filter($"a" % 2 === 0)

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.where(orders.a == 'red')

Or

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.filter($("b").isEqual("red"));

Scala

  1. val orders: Table = tableEnv.from("Orders")
  2. val result = orders.filter($"a" % 2 === 0)

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.filter(orders.a == 'red')

Column Operations

AddColumns

Batch Streaming

Performs a field add operation. It will throw an exception if the added fields already exist.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.addColumns(concat($("c"), "sunny"));

Scala

  1. val orders = tableEnv.from("Orders");
  2. val result = orders.addColumns(concat($"c", "Sunny"))

Python

  1. from pyflink.table.expressions import concat
  2. orders = t_env.from_path("Orders")
  3. result = orders.add_columns(concat(orders.c, 'sunny'))

AddOrReplaceColumns

Batch Streaming

Performs a field add operation. Existing fields will be replaced if the added column name is the same as the existing column name. Moreover, if the added fields have duplicate field name, then the last one is used.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.addOrReplaceColumns(concat($("c"), "sunny").as("desc"));

Scala

  1. val orders = tableEnv.from("Orders");
  2. val result = orders.addOrReplaceColumns(concat($"c", "Sunny") as "desc")

Python

  1. from pyflink.table.expressions import concat
  2. orders = t_env.from_path("Orders")
  3. result = orders.add_or_replace_columns(concat(orders.c, 'sunny').alias('desc'))

DropColumns

Batch Streaming

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.dropColumns($("b"), $("c"));

Scala

  1. val orders = tableEnv.from("Orders");
  2. val result = orders.dropColumns($"b", $"c")

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.drop_columns(orders.b, orders.c)

RenameColumns

Batch Streaming

Performs a field rename operation. The field expressions should be alias expressions, and only the existing fields can be renamed.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.renameColumns($("b").as("b2"), $("c").as("c2"));

Scala

  1. val orders = tableEnv.from("Orders");
  2. val result = orders.renameColumns($"b" as "b2", $"c" as "c2")

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.rename_columns(orders.b.alias('b2'), orders.c.alias('c2'))

Aggregations

GroupBy Aggregation

Batch Streaming Result Updating

Similar to a SQL GROUP BY clause. Groups the rows on the grouping keys with a following running aggregation operator to aggregate rows group-wise.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.groupBy($("a")).select($("a"), $("b").sum().as("d"));

Scala

  1. val orders: Table = tableEnv.from("Orders")
  2. val result = orders.groupBy($"a").select($"a", $"b".sum().as("d"))

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.group_by(orders.a).select(orders.a, orders.b.sum.alias('d'))

For streaming queries the required state to compute the query result might grow infinitely depending on the type of aggregation and the number of distinct grouping keys. Please provide a query configuration with valid retention interval to prevent excessive state size. See Query Configuration for details.

GroupBy Window Aggregation

Batch Streaming

Groups and aggregates a table on a group window and possibly one or more grouping keys.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders
  3. .window(Tumble.over(lit(5).minutes()).on($("rowtime")).as("w")) // define window
  4. .groupBy($("a"), $("w")) // group by key and window
  5. // access window properties and aggregate
  6. .select(
  7. $("a"),
  8. $("w").start(),
  9. $("w").end(),
  10. $("w").rowtime(),
  11. $("b").sum().as("d")
  12. );

Scala

  1. val orders: Table = tableEnv.from("Orders")
  2. val result: Table = orders
  3. .window(Tumble over 5.minutes on $"rowtime" as "w") // define window
  4. .groupBy($"a", $"w") // group by key and window
  5. .select($"a", $"w".start, $"w".end, $"w".rowtime, $"b".sum as "d") // access window properties and aggregate

Python

  1. from pyflink.table.window import Tumble
  2. from pyflink.table.expressions import lit, col
  3. orders = t_env.from_path("Orders")
  4. result = orders.window(Tumble.over(lit(5).minutes).on(orders.rowtime).alias("w")) \
  5. .group_by(orders.a, col('w')) \
  6. .select(orders.a, col('w').start, col('w').end, orders.b.sum.alias('d'))

Over Window Aggregation

Similar to a SQL OVER clause. Over window aggregates are computed for each row, based on a window (range) of preceding and succeeding rows. See the over windows section for more details.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders
  3. // define window
  4. .window(
  5. Over
  6. .partitionBy($("a"))
  7. .orderBy($("rowtime"))
  8. .preceding(UNBOUNDED_RANGE)
  9. .following(CURRENT_RANGE)
  10. .as("w"))
  11. // sliding aggregate
  12. .select(
  13. $("a"),
  14. $("b").avg().over($("w")),
  15. $("b").max().over($("w")),
  16. $("b").min().over($("w"))
  17. );

Scala

  1. val orders: Table = tableEnv.from("Orders")
  2. val result: Table = orders
  3. // define window
  4. .window(
  5. Over
  6. partitionBy $"a"
  7. orderBy $"rowtime"
  8. preceding UNBOUNDED_RANGE
  9. following CURRENT_RANGE
  10. as "w")
  11. .select($"a", $"b".avg over $"w", $"b".max().over($"w"), $"b".min().over($"w")) // sliding aggregate

Python

  1. from pyflink.table.window import Over
  2. from pyflink.table.expressions import col, UNBOUNDED_RANGE, CURRENT_RANGE
  3. orders = t_env.from_path("Orders")
  4. result = orders.over_window(Over.partition_by(orders.a).order_by(orders.rowtime)
  5. .preceding(UNBOUNDED_RANGE).following(CURRENT_RANGE)
  6. .alias("w")) \
  7. .select(orders.a, orders.b.avg.over(col('w')), orders.b.max.over(col('w')), orders.b.min.over(col('w')))

All aggregates must be defined over the same window, i.e., same partitioning, sorting, and range. Currently, only windows with PRECEDING (UNBOUNDED and bounded) to CURRENT ROW range are supported. Ranges with FOLLOWING are not supported yet. ORDER BY must be specified on a single time attribute.

Distinct Aggregation

Batch Streaming Result Updating

Similar to a SQL DISTINCT aggregation clause such as COUNT(DISTINCT a). Distinct aggregation declares that an aggregation function (built-in or user-defined) is only applied on distinct input values. Distinct can be applied to GroupBy Aggregation, GroupBy Window Aggregation and Over Window Aggregation.

Java

  1. Table orders = tableEnv.from("Orders");
  2. // Distinct aggregation on group by
  3. Table groupByDistinctResult = orders
  4. .groupBy($("a"))
  5. .select($("a"), $("b").sum().distinct().as("d"));
  6. // Distinct aggregation on time window group by
  7. Table groupByWindowDistinctResult = orders
  8. .window(Tumble
  9. .over(lit(5).minutes())
  10. .on($("rowtime"))
  11. .as("w")
  12. )
  13. .groupBy($("a"), $("w"))
  14. .select($("a"), $("b").sum().distinct().as("d"));
  15. // Distinct aggregation on over window
  16. Table result = orders
  17. .window(Over
  18. .partitionBy($("a"))
  19. .orderBy($("rowtime"))
  20. .preceding(UNBOUNDED_RANGE)
  21. .as("w"))
  22. .select(
  23. $("a"), $("b").avg().distinct().over($("w")),
  24. $("b").max().over($("w")),
  25. $("b").min().over($("w"))
  26. );

Scala

  1. val orders: Table = tableEnv.from("Orders");
  2. // Distinct aggregation on group by
  3. val groupByDistinctResult = orders
  4. .groupBy($"a")
  5. .select($"a", $"b".sum.distinct as "d")
  6. // Distinct aggregation on time window group by
  7. val groupByWindowDistinctResult = orders
  8. .window(Tumble over 5.minutes on $"rowtime" as "w").groupBy($"a", $"w")
  9. .select($"a", $"b".sum.distinct as "d")
  10. // Distinct aggregation on over window
  11. val result = orders
  12. .window(Over
  13. partitionBy $"a"
  14. orderBy $"rowtime"
  15. preceding UNBOUNDED_RANGE
  16. as $"w")
  17. .select($"a", $"b".avg.distinct over $"w", $"b".max over $"w", $"b".min over $"w")

Python

  1. from pyflink.table.expressions import col, lit, UNBOUNDED_RANGE
  2. orders = t_env.from_path("Orders")
  3. # Distinct aggregation on group by
  4. group_by_distinct_result = orders.group_by(orders.a) \
  5. .select(orders.a, orders.b.sum.distinct.alias('d'))
  6. # Distinct aggregation on time window group by
  7. group_by_window_distinct_result = orders.window(
  8. Tumble.over(lit(5).minutes).on(orders.rowtime).alias("w")).group_by(orders.a, col('w')) \
  9. .select(orders.a, orders.b.sum.distinct.alias('d'))
  10. # Distinct aggregation on over window
  11. result = orders.over_window(Over
  12. .partition_by(orders.a)
  13. .order_by(orders.rowtime)
  14. .preceding(UNBOUNDED_RANGE)
  15. .alias("w")) \
  16. .select(orders.a, orders.b.avg.distinct.over(col('w')), orders.b.max.over(col('w')), orders.b.min.over(col('w')))

User-defined aggregation function can also be used with DISTINCT modifiers. To calculate the aggregate results only for distinct values, simply add the distinct modifier towards the aggregation function.

Java

  1. Table orders = tEnv.from("Orders");
  2. // Use distinct aggregation for user-defined aggregate functions
  3. tEnv.registerFunction("myUdagg", new MyUdagg());
  4. orders.groupBy("users")
  5. .select(
  6. $("users"),
  7. call("myUdagg", $("points")).distinct().as("myDistinctResult")
  8. );

Scala

  1. val orders: Table = tEnv.from("Orders");
  2. // Use distinct aggregation for user-defined aggregate functions
  3. val myUdagg = new MyUdagg();
  4. orders.groupBy($"users").select($"users", myUdagg.distinct($"points") as "myDistinctResult");

Python

Unsupported

For streaming queries the required state to compute the query result might grow infinitely depending on the type of aggregation and the number of distinct grouping keys. Please provide a query configuration with valid retention interval to prevent excessive state size. See Query Configuration for details.

Distinct

Batch Streaming Result Updating

Similar to a SQL DISTINCT clause. Returns records with distinct value combinations.

Java

  1. Table orders = tableEnv.from("Orders");
  2. Table result = orders.distinct();

Scala

  1. val orders: Table = tableEnv.from("Orders")
  2. val result = orders.distinct()

Python

  1. orders = t_env.from_path("Orders")
  2. result = orders.distinct()

For streaming queries the required state to compute the query result might grow infinitely depending on the type of aggregation and the number of distinct grouping keys. Please provide a query configuration with valid retention interval to prevent excessive state size. See Query Configuration for details.

Joins

Inner Join

Batch Streaming

Similar to a SQL JOIN clause. Joins two tables. Both tables must have distinct field names and at least one equality join predicate must be defined through join operator or using a where or filter operator.

Java

  1. Table left = tableEnv.fromDataSet(ds1, "a, b, c");
  2. Table right = tableEnv.fromDataSet(ds2, "d, e, f");
  3. Table result = left.join(right)
  4. .where($("a").isEqual($("d")))
  5. .select($("a"), $("b"), $("e"));

Scala

  1. val left = ds1.toTable(tableEnv, $"a", $"b", $"c")
  2. val right = ds2.toTable(tableEnv, $"d", $"e", $"f")
  3. val result = left.join(right).where($"a" === $"d").select($"a", $"b", $"e")

Python

  1. from pyflink.table.expressions import col
  2. left = t_env.from_path("Source1").select(col('a'), col('b'), col('c'))
  3. right = t_env.from_path("Source2").select(col('d'), col('e'), col('f'))
  4. result = left.join(right).where(left.a == right.d).select(left.a, left.b, right.e)

For streaming queries the required state to compute the query result might grow infinitely depending on the type of aggregation and the number of distinct grouping keys. Please provide a query configuration with valid retention interval to prevent excessive state size. See Query Configuration for details.

Outer Join

Batch Streaming Result Updating

Similar to SQL LEFT/RIGHT/FULL OUTER JOIN clauses. Joins two tables. Both tables must have distinct field names and at least one equality join predicate must be defined.

Java

  1. Table left = tableEnv.fromDataSet(ds1, "a, b, c");
  2. Table right = tableEnv.fromDataSet(ds2, "d, e, f");
  3. Table leftOuterResult = left.leftOuterJoin(right, $("a").isEqual($("d")))
  4. .select($("a"), $("b"), $("e"));
  5. Table rightOuterResult = left.rightOuterJoin(right, $("a").isEqual($("d")))
  6. .select($("a"), $("b"), $("e"));
  7. Table fullOuterResult = left.fullOuterJoin(right, $("a").isEqual($("d")))
  8. .select($("a"), $("b"), $("e"));

Scala

  1. val left = tableEnv.fromDataSet(ds1, $"a", $"b", $"c")
  2. val right = tableEnv.fromDataSet(ds2, $"d", $"e", $"f")
  3. val leftOuterResult = left.leftOuterJoin(right, $"a" === $"d").select($"a", $"b", $"e")
  4. val rightOuterResult = left.rightOuterJoin(right, $"a" === $"d").select($"a", $"b", $"e")
  5. val fullOuterResult = left.fullOuterJoin(right, $"a" === $"d").select($"a", $"b", $"e")

Python

  1. from pyflink.table.expressions import col
  2. left = t_env.from_path("Source1").select(col('a'), col('b'), col('c'))
  3. right = t_env.from_path("Source2").select(col('d'), col('e'), col('f'))
  4. left_outer_result = left.left_outer_join(right, left.a == right.d).select(left.a, left.b, right.e)
  5. right_outer_result = left.right_outer_join(right, left.a == right.d).select(left.a, left.b, right.e)
  6. full_outer_result = left.full_outer_join(right, left.a == right.d).select(left.a, left.b, right.e)

For streaming queries the required state to compute the query result might grow infinitely depending on the type of aggregation and the number of distinct grouping keys. Please provide a query configuration with valid retention interval to prevent excessive state size. See Query Configuration for details.

Interval Join

Batch Streaming

Interval joins are a subset of regular joins that can be processed in a streaming fashion.

An interval join requires at least one equi-join predicate and a join condition that bounds the time on both sides. Such a condition can be defined by two appropriate range predicates (<, <=, >=, >) or a single equality predicate that compares time attributes of the same type (i.e., processing time or event time) of both input tables.

Java

  1. Table left = tableEnv.fromDataSet(ds1, $("a"), $("b"), $("c"), $("ltime").rowtime());
  2. Table right = tableEnv.fromDataSet(ds2, $("d"), $("e"), $("f"), $("rtime").rowtime()));
  3. Table result = left.join(right)
  4. .where(
  5. and(
  6. $("a").isEqual($("d")),
  7. $("ltime").isGreaterOrEqual($("rtime").minus(lit(5).minutes())),
  8. $("ltime").isLess($("rtime").plus(lit(10).minutes()))
  9. ))
  10. .select($("a"), $("b"), $("e"), $("ltime"));

Scala

  1. val left = ds1.toTable(tableEnv, $"a", $"b", $"c", $"ltime".rowtime)
  2. val right = ds2.toTable(tableEnv, $"d", $"e", $"f", $"rtime".rowtime)
  3. val result = left.join(right)
  4. .where($"a" === $"d" && $"ltime" >= $"rtime" - 5.minutes && $"ltime" < $"rtime" + 10.minutes)
  5. .select($"a", $"b", $"e", $"ltime")

Python

  1. from pyflink.table.expressions import col
  2. left = t_env.from_path("Source1").select(col('a'), col('b'), col('c'), col('rowtime1'))
  3. right = t_env.from_path("Source2").select(col('d'), col('e'), col('f'), col('rowtime2'))
  4. joined_table = left.join(right).where(left.a == right.d & left.rowtime1 >= right.rowtime2 - lit(1).second & left.rowtime1 <= right.rowtime2 + lit(2).seconds)
  5. result = joined_table.select(joined_table.a, joined_table.b, joined_table.e, joined_table.rowtime1)

Inner Join with Table Function (UDTF)

Batch Streaming

Joins a table with the results of a table function. Each row of the left (outer) table is joined with all rows produced by the corresponding call of the table function. A row of the left (outer) table is dropped, if its table function call returns an empty result.

Java

  1. // register User-Defined Table Function
  2. TableFunction<Tuple3<String,String,String>> split = new MySplitUDTF();
  3. tableEnv.registerFunction("split", split);
  4. // join
  5. Table orders = tableEnv.from("Orders");
  6. Table result = orders
  7. .joinLateral(call("split", $("c")).as("s", "t", "v"))
  8. .select($("a"), $("b"), $("s"), $("t"), $("v"));

Scala

  1. // instantiate User-Defined Table Function
  2. val split: TableFunction[_] = new MySplitUDTF()
  3. // join
  4. val result: Table = table
  5. .joinLateral(split($"c") as ("s", "t", "v"))
  6. .select($"a", $"b", $"s", $"t", $"v")

Python

  1. # register User-Defined Table Function
  2. @udtf(result_types=[DataTypes.BIGINT(), DataTypes.BIGINT(), DataTypes.BIGINT()])
  3. def split(x):
  4. return [Row(1, 2, 3)]
  5. # join
  6. orders = t_env.from_path("Orders")
  7. joined_table = orders.join_lateral(split(orders.c).alias("s, t, v"))
  8. result = joined_table.select(joined_table.a, joined_table.b, joined_table.s, joined_table.t, joined_table.v)

Left Outer Join with Table Function (UDTF)

Batch Streaming

Joins a table with the results of a table function. Each row of the left (outer) table is joined with all rows produced by the corresponding call of the table function. If a table function call returns an empty result, the corresponding outer row is preserved and the result padded with null values.

Currently, the predicate of a table function left outer join can only be empty or literal true.

Java

  1. // register User-Defined Table Function
  2. TableFunction<Tuple3<String,String,String>> split = new MySplitUDTF();
  3. tableEnv.registerFunction("split", split);
  4. // join
  5. Table orders = tableEnv.from("Orders");
  6. Table result = orders
  7. .leftOuterJoinLateral(call("split", $("c")).as("s", "t", "v"))
  8. .select($("a"), $("b"), $("s"), $("t"), $("v"));

Scala

  1. // instantiate User-Defined Table Function
  2. val split: TableFunction[_] = new MySplitUDTF()
  3. // join
  4. val result: Table = table
  5. .leftOuterJoinLateral(split($"c") as ("s", "t", "v"))
  6. .select($"a", $"b", $"s", $"t", $"v")

Python

  1. # register User-Defined Table Function
  2. @udtf(result_types=[DataTypes.BIGINT(), DataTypes.BIGINT(), DataTypes.BIGINT()])
  3. def split(x):
  4. return [Row(1, 2, 3)]
  5. # join
  6. orders = t_env.from_path("Orders")
  7. joined_table = orders.left_outer_join_lateral(split(orders.c).alias("s, t, v"))
  8. result = joined_table.select(joined_table.a, joined_table.b, joined_table.s, joined_table.t, joined_table.v)

Join with Temporal TAble

Temporal tables are tables that track changes over time.

A temporal table function provides access to the state of a temporal table at a specific point in time. The syntax to join a table with a temporal table function is the same as in Inner Join with Table Function.

Currently only inner joins with temporal tables are supported.

Java

  1. Table ratesHistory = tableEnv.from("RatesHistory");
  2. // register temporal table function with a time attribute and primary key
  3. TemporalTableFunction rates = ratesHistory.createTemporalTableFunction(
  4. "r_proctime",
  5. "r_currency");
  6. tableEnv.registerFunction("rates", rates);
  7. // join with "Orders" based on the time attribute and key
  8. Table orders = tableEnv.from("Orders");
  9. Table result = orders
  10. .joinLateral(call("rates", $("o_proctime")), $("o_currency").isEqual($("r_currency")));

Python

Currently not supported in Python Table API.

Set Operations

Union

Batch

Similar to a SQL UNION clause. Unions two tables with duplicate records removed. Both tables must have identical field types.

Java

  1. Table left = tableEnv.from("orders1");
  2. Table right = tableEnv.from("orders2");
  3. left.union(right);

Scala

  1. val left = tableEnv.from("orders1")
  2. val right = tableEnv.from("orders2")
  3. left.union(right)

left = tableEnv.from_path(“orders1”) right = tableEnv.from_path(“orders2”)

left.union(right)

UnionAll

Batch Streaming

Similar to a SQL UNION ALL clause. Unions two tables. Both tables must have identical field types.

Java

  1. Table left = tableEnv.from("orders1");
  2. Table right = tableEnv.from("orders2");
  3. left.unionAl(right);

Scala

  1. val left = tableEnv.from("orders1")
  2. val right = tableEnv.from("orders2")
  3. left.unionAl(right)

left = tableEnv.from_path(“orders1”) right = tableEnv.from_path(“orders2”)

left.unionAl(right)

Intersect

Batch

Similar to a SQL INTERSECT clause. Intersect returns records that exist in both tables. If a record is present one or both tables more than once, it is returned just once, i.e., the resulting table has no duplicate records. Both tables must have identical field types.

Java

  1. Table left = tableEnv.from("orders1");
  2. Table right = tableEnv.from("orders2");
  3. left.intersect(right);

Scala

  1. val left = tableEnv.from("orders1")
  2. val right = tableEnv.from("orders2")
  3. left.intersect(right)

left = tableEnv.from_path(“orders1”) right = tableEnv.from_path(“orders2”)

left.intersect(right)

IntersectAll

Batch

Similar to a SQL INTERSECT ALL clause. IntersectAll returns records that exist in both tables. If a record is present in both tables more than once, it is returned as many times as it is present in both tables, i.e., the resulting table might have duplicate records. Both tables must have identical field types.

Java

  1. Table left = tableEnv.from("orders1");
  2. Table right = tableEnv.from("orders2");
  3. left.intersectAll(right);

Scala

  1. val left = tableEnv.from("orders1")
  2. val right = tableEnv.from("orders2")
  3. left.intersectAll(right)

left = tableEnv.from_path(“orders1”) right = tableEnv.from_path(“orders2”)

left.intersectAll(right)

Minus

Batch

Similar to a SQL EXCEPT clause. Minus returns records from the left table that do not exist in the right table. Duplicate records in the left table are returned exactly once, i.e., duplicates are removed. Both tables must have identical field types.

Java

  1. Table left = tableEnv.from("orders1");
  2. Table right = tableEnv.from("orders2");
  3. left.minus(right);

Scala

  1. val left = tableEnv.from("orders1")
  2. val right = tableEnv.from("orders2")
  3. left.minus(right)

left = tableEnv.from_path(“orders1”) right = tableEnv.from_path(“orders2”)

left.minus(right)

MinusAll

Batch

Similar to a SQL EXCEPT ALL clause. MinusAll returns the records that do not exist in the right table. A record that is present n times in the left table and m times in the right table is returned (n - m) times, i.e., as many duplicates as are present in the right table are removed. Both tables must have identical field types.

Java

  1. Table left = tableEnv.from("orders1");
  2. Table right = tableEnv.from("orders2");
  3. left.minusAll(right);

Scala

  1. val left = tableEnv.from("orders1")
  2. val right = tableEnv.from("orders2")
  3. left.minusAll(right)

left = tableEnv.from_path(“orders1”) right = tableEnv.from_path(“orders2”)

left.minusAll(right)

In

Batch Streaming

Similar to a SQL IN clause. In returns true if an expression exists in a given table sub-query. The sub-query table must consist of one column. This column must have the same data type as the expression.

Java

  1. Table left = tableEnv.from("Orders1")
  2. Table right = tableEnv.from("Orders2");
  3. Table result = left.select($("a"), $("b"), $("c")).where($("a").in(right));

Scala

  1. val left = tableEnv.from("Orders1")
  2. val right = tableEnv.from("Orders2");
  3. val result = left.select($"a", $"b", $"c").where($"a".in(right))

Python

  1. left = t_env.from_path("Source1").select(col('a'), col('b'), col('c'))
  2. right = t_env.from_path("Source2").select(col('a'))
  3. result = left.select(left.a, left.b, left.c).where(left.a.in_(right))

For streaming queries the required state to compute the query result might grow infinitely depending on the type of aggregation and the number of distinct grouping keys. Please provide a query configuration with valid retention interval to prevent excessive state size. See Query Configuration for details.

OrderBy, Offset & Fetch

Order By

Batch Streaming

Similar to a SQL ORDER BY clause. Returns records globally sorted across all parallel partitions. For unbounded tables, this operation requires a sorting on a time attribute or a subsequent fetch operation.

Java

  1. Table result = in.orderBy($("a").asc());

Scala

  1. val result = in.orderBy($"a".asc)

Python

  1. result = in.order_by(in.a.asc)

Offset & Fetch

Batch Streaming

Similar to the SQL OFFSET and FETCH clauses. The offset operation limits a (possibly sorted) result from an offset position. The fetch operation limits a (possibly sorted) result to the first n rows. Usually, the two operations are preceded by an ordering operator. For unbounded tables, a fetch operation is required for an offset operation.

Java

  1. // returns the first 5 records from the sorted result
  2. Table result1 = in.orderBy($("a").asc()).fetch(5);
  3. // skips the first 3 records and returns all following records from the sorted result
  4. Table result2 = in.orderBy($("a").asc()).offset(3);
  5. // skips the first 10 records and returns the next 5 records from the sorted result
  6. Table result3 = in.orderBy($("a").asc()).offset(10).fetch(5);

Scala

  1. // returns the first 5 records from the sorted result
  2. val result1: Table = in.orderBy($"a".asc).fetch(5)
  3. // skips the first 3 records and returns all following records from the sorted result
  4. val result2: Table = in.orderBy($"a".asc).offset(3)
  5. // skips the first 10 records and returns the next 5 records from the sorted result
  6. val result3: Table = in.orderBy($"a".asc).offset(10).fetch(5)

Python

  1. # returns the first 5 records from the sorted result
  2. result1 = table.order_by(table.a.asc).fetch(5)
  3. # skips the first 3 records and returns all following records from the sorted result
  4. result2 = table.order_by(table.a.asc).offset(3)
  5. # skips the first 10 records and returns the next 5 records from the sorted result
  6. result3 = table.order_by(table.a.asc).offset(10).fetch(5)

Insert

Batch Streaming

Similar to the INSERT INTO clause in a SQL query, the method performs an insertion into a registered output table. The executeInsert() method will immediately submit a Flink job which execute the insert operation.

Output tables must be registered in the TableEnvironment (see Connector tables). Moreover, the schema of the registered table must match the schema of the query.

Java

  1. Table orders = tableEnv.from("Orders");
  2. orders.executeInsert("OutOrders");

Scala

  1. val orders = tableEnv.from("Orders")
  2. orders.executeInsert("OutOrders")

Python

  1. orders = t_env.from_path("Orders")
  2. orders.execute_insert("OutOrders")

Group Windows

Group window aggregates group rows into finite groups based on time or row-count intervals and evaluate aggregation functions once per group. For batch tables, windows are a convenient shortcut to group records by time intervals.

Java

Windows are defined using the window(GroupWindow w) clause and require an alias, which is specified using the as clause. In order to group a table by a window, the window alias must be referenced in the groupBy(...) clause like a regular grouping attribute. The following example shows how to define a window aggregation on a table.

  1. Table table = input
  2. .window([GroupWindow w].as("w")) // define window with alias w
  3. .groupBy($("w")) // group the table by window w
  4. .select($("b").sum()); // aggregate

Scala

Windows are defined using the window(w: GroupWindow) clause and require an alias, which is specified using the as clause. In order to group a table by a window, the window alias must be referenced in the groupBy(...) clause like a regular grouping attribute. The following example shows how to define a window aggregation on a table.

  1. val table = input
  2. .window([w: GroupWindow] as $"w") // define window with alias w
  3. .groupBy($"w") // group the table by window w
  4. .select($"b".sum) // aggregate

Python

Windows are defined using the window(w: GroupWindow) clause and require an alias, which is specified using the alias clause. In order to group a table by a window, the window alias must be referenced in the group_by(...) clause like a regular grouping attribute. The following example shows how to define a window aggregation on a table.

  1. # define window with alias w, group the table by window w, then aggregate
  2. table = input.window([w: GroupWindow].alias("w")) \
  3. .group_by(col('w')).select(input.b.sum)

Java

In streaming environments, window aggregates can only be computed in parallel if they group on one or more attributes in addition to the window, i.e., the groupBy(...) clause references a window alias and at least one additional attribute. A groupBy(...) clause that only references a window alias (such as in the example above) can only be evaluated by a single, non-parallel task. The following example shows how to define a window aggregation with additional grouping attributes.

  1. Table table = input
  2. .window([GroupWindow w].as("w")) // define window with alias w
  3. .groupBy($("w"), $("a")) // group the table by attribute a and window w
  4. .select($("a"), $("b").sum()); // aggregate

Scala

In streaming environments, window aggregates can only be computed in parallel if they group on one or more attributes in addition to the window, i.e., the groupBy(...) clause references a window alias and at least one additional attribute. A groupBy(...) clause that only references a window alias (such as in the example above) can only be evaluated by a single, non-parallel task. The following example shows how to define a window aggregation with additional grouping attributes.

  1. val table = input
  2. .window([w: GroupWindow] as $"w") // define window with alias w
  3. .groupBy($"w", $"a") // group the table by attribute a and window w
  4. .select($"a", $"b".sum) // aggregate

Python

In streaming environments, window aggregates can only be computed in parallel if they group on one or more attributes in addition to the window, i.e., the group_by(...) clause references a window alias and at least one additional attribute. A group_by(...) clause that only references a window alias (such as in the example above) can only be evaluated by a single, non-parallel task. The following example shows how to define a window aggregation with additional grouping attributes.

  1. # define window with alias w, group the table by attribute a and window w,
  2. # then aggregate
  3. table = input.window([w: GroupWindow].alias("w")) \
  4. .group_by(col('w'), input.a).select(input.b.sum)

Window properties such as the start, end, or rowtime timestamp of a time window can be added in the select statement as a property of the window alias as w.start, w.end, and w.rowtime, respectively. The window start and rowtime timestamps are the inclusive lower and upper window boundaries. In contrast, the window end timestamp is the exclusive upper window boundary. For example a tumbling window of 30 minutes that starts at 2pm would have 14:00:00.000 as start timestamp, 14:29:59.999 as rowtime timestamp, and 14:30:00.000 as end timestamp.

Java

  1. Table table = input
  2. .window([GroupWindow w].as("w")) // define window with alias w
  3. .groupBy($("w"), $("a")) // group the table by attribute a and window w
  4. .select($("a"), $("w").start(), $("w").end(), $("w").rowtime(), $("b").count()); // aggregate and add window start, end, and rowtime timestamps

Scala

  1. val table = input
  2. .window([w: GroupWindow] as $"w") // define window with alias w
  3. .groupBy($"w", $"a") // group the table by attribute a and window w
  4. .select($"a", $"w".start, $"w".end, $"w".rowtime, $"b".count) // aggregate and add window start, end, and rowtime timestamps

Python

  1. # define window with alias w, group the table by attribute a and window w,
  2. # then aggregate and add window start, end, and rowtime timestamps
  3. table = input.window([w: GroupWindow].alias("w")) \
  4. .group_by(col('w'), input.a) \
  5. .select(input.a, col('w').start, col('w').end, col('w').rowtime, input.b.count)

The Window parameter defines how rows are mapped to windows. Window is not an interface that users can implement. Instead, the Table API provides a set of predefined Window classes with specific semantics, which are translated into underlying DataStream or DataSet operations. The supported window definitions are listed below.

Tumble (Tumbling Windows)

A tumbling window assigns rows to non-overlapping, continuous windows of fixed length. For example, a tumbling window of 5 minutes groups rows in 5 minutes intervals. Tumbling windows can be defined on event-time, processing-time, or on a row-count.

Java

Tumbling windows are defined by using the Tumble class as follows:

MethodDescription
overDefines the length the window, either as time or row-count interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
asAssigns an alias to the window. The alias is used to reference the window in the following groupBy() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. // Tumbling Event-time Window
  2. .window(Tumble.over(lit(10).minutes()).on($("rowtime")).as("w"));
  3. // Tumbling Processing-time Window (assuming a processing-time attribute "proctime")
  4. .window(Tumble.over(lit(10).minutes()).on($("proctime")).as("w"));
  5. // Tumbling Row-count Window (assuming a processing-time attribute "proctime")
  6. .window(Tumble.over(rowInterval(10)).on($("proctime")).as("w"));

Scala

Tumbling windows are defined by using the Tumble class as follows:

MethodDescription
overDefines the length the window, either as time or row-count interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
asAssigns an alias to the window. The alias is used to reference the window in the following groupBy() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. // Tumbling Event-time Window
  2. .window(Tumble over 10.minutes on $"rowtime" as $"w")
  3. // Tumbling Processing-time Window (assuming a processing-time attribute "proctime")
  4. .window(Tumble over 10.minutes on $"proctime" as $"w")
  5. // Tumbling Row-count Window (assuming a processing-time attribute "proctime")
  6. .window(Tumble over 10.rows on $"proctime" as $"w")

Python

Tumbling windows are defined by using the Tumble class as follows:

MethodDescription
overDefines the length the window, either as time or row-count interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
aliasAssigns an alias to the window. The alias is used to reference the window in the following group_by() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. # Tumbling Event-time Window
  2. .window(Tumble.over(lit(10).minutes).on(col('rowtime')).alias("w"))
  3. # Tumbling Processing-time Window (assuming a processing-time attribute "proctime")
  4. .window(Tumble.over(lit(10).minutes).on(col('proctime')).alias("w"))
  5. # Tumbling Row-count Window (assuming a processing-time attribute "proctime")
  6. .window(Tumble.over(row_interval(10)).on(col('proctime')).alias("w"))

Slide (Sliding Windows)

A sliding window has a fixed size and slides by a specified slide interval. If the slide interval is smaller than the window size, sliding windows are overlapping. Thus, rows can be assigned to multiple windows. For example, a sliding window of 15 minutes size and 5 minute slide interval assigns each row to 3 different windows of 15 minute size, which are evaluated in an interval of 5 minutes. Sliding windows can be defined on event-time, processing-time, or on a row-count.

Java

Sliding windows are defined by using the Slide class as follows:

MethodDescription
overDefines the length of the window, either as time or row-count interval.
everyDefines the slide interval, either as time or row-count interval. The slide interval must be of the same type as the size interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
asAssigns an alias to the window. The alias is used to reference the window in the following groupBy() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. // Sliding Event-time Window
  2. .window(Slide.over(lit(10).minutes())
  3. .every(lit(5).minutes())
  4. .on($("rowtime"))
  5. .as("w"));
  6. // Sliding Processing-time window (assuming a processing-time attribute "proctime")
  7. .window(Slide.over(lit(10).minutes())
  8. .every(lit(5).minutes())
  9. .on($("proctime"))
  10. .as("w"));
  11. // Sliding Row-count window (assuming a processing-time attribute "proctime")
  12. .window(Slide.over(rowInterval(10)).every(rowInterval(5)).on($("proctime")).as("w"));

Scala

Sliding windows are defined by using the Slide class as follows:

MethodDescription
overDefines the length of the window, either as time or row-count interval.
everyDefines the slide interval, either as time or row-count interval. The slide interval must be of the same type as the size interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
asAssigns an alias to the window. The alias is used to reference the window in the following groupBy() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. // Sliding Event-time Window
  2. .window(Slide over 10.minutes every 5.minutes on $"rowtime" as $"w")
  3. // Sliding Processing-time window (assuming a processing-time attribute "proctime")
  4. .window(Slide over 10.minutes every 5.minutes on $"proctime" as $"w")
  5. // Sliding Row-count window (assuming a processing-time attribute "proctime")
  6. .window(Slide over 10.rows every 5.rows on $"proctime" as $"w")

Python

Sliding windows are defined by using the Slide class as follows:

MethodDescription
overDefines the length of the window, either as time or row-count interval.
everyDefines the slide interval, either as time or row-count interval. The slide interval must be of the same type as the size interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
aliasAssigns an alias to the window. The alias is used to reference the window in the following group_by() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. # Sliding Event-time Window
  2. .window(Slide.over(lit(10).minutes).every(lit(5).minutes).on(col('rowtime')).alias("w"))
  3. # Sliding Processing-time window (assuming a processing-time attribute "proctime")
  4. .window(Slide.over(lit(10).minutes).every(lit(5).minutes).on(col('proctime')).alias("w"))
  5. # Sliding Row-count window (assuming a processing-time attribute "proctime")
  6. .window(Slide.over(row_interval(10)).every(row_interval(5)).on(col('proctime')).alias("w"))

Session (Session Windows)

Session windows do not have a fixed size but their bounds are defined by an interval of inactivity, i.e., a session window is closes if no event appears for a defined gap period. For example a session window with a 30 minute gap starts when a row is observed after 30 minutes inactivity (otherwise the row would be added to an existing window) and is closed if no row is added within 30 minutes. Session windows can work on event-time or processing-time.

Java

A session window is defined by using the Session class as follows:

MethodDescription
withGapDefines the gap between two windows as time interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
asAssigns an alias to the window. The alias is used to reference the window in the following groupBy() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. // Session Event-time Window
  2. .window(Session.withGap(lit(10).minutes()).on($("rowtime")).as("w"));
  3. // Session Processing-time Window (assuming a processing-time attribute "proctime")
  4. .window(Session.withGap(lit(10).minutes()).on($("proctime")).as("w"));

Scala

A session window is defined by using the Session class as follows:

MethodDescription
withGapDefines the gap between two windows as time interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
asAssigns an alias to the window. The alias is used to reference the window in the following groupBy() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. // Session Event-time Window
  2. .window(Session withGap 10.minutes on $"rowtime" as $"w")
  3. // Session Processing-time Window (assuming a processing-time attribute "proctime")
  4. .window(Session withGap 10.minutes on $"proctime" as $"w")

Python

A session window is defined by using the Session class as follows:

MethodDescription
with_gapDefines the gap between two windows as time interval.
onThe time attribute to group (time interval) or sort (row count) on. For batch queries this might be any Long or Timestamp attribute. For streaming queries this must be a declared event-time or processing-time time attribute.
aliasAssigns an alias to the window. The alias is used to reference the window in the following group_by() clause and optionally to select window properties such as window start, end, or rowtime timestamps in the select() clause.
  1. # Session Event-time Window
  2. .window(Session.with_gap(lit(10).minutes).on(col('rowtime')).alias("w"))
  3. # Session Processing-time Window (assuming a processing-time attribute "proctime")
  4. .window(Session.with_gap(lit(10).minutes).on(col('proctime')).alias("w"))

Over Windows

Over window aggregates are known from standard SQL (OVER clause) and defined in the SELECT clause of a query. Unlike group windows, which are specified in the GROUP BY clause, over windows do not collapse rows. Instead over window aggregates compute an aggregate for each input row over a range of its neighboring rows.

Over windows are defined using the window(w: OverWindow*) clause (using over_window(*OverWindow) in Python API) and referenced via an alias in the select() method. The following example shows how to define an over window aggregation on a table.

Java

  1. Table table = input
  2. .window([OverWindow w].as("w")) // define over window with alias w
  3. .select($("a"), $("b").sum().over($("w")), $("c").min().over($("w"))); // aggregate over the over window w

Scala

  1. val table = input
  2. .window([w: OverWindow] as $"w") // define over window with alias w
  3. .select($"a", $"b".sum over $"w", $"c".min over $"w") // aggregate over the over window w

Python

  1. # define over window with alias w and aggregate over the over window w
  2. table = input.over_window([w: OverWindow].alias("w")) \
  3. .select(input.a, input.b.sum.over(col('w')), input.c.min.over(col('w')))

The OverWindow defines a range of rows over which aggregates are computed. OverWindow is not an interface that users can implement. Instead, the Table API provides the Over class to configure the properties of the over window. Over windows can be defined on event-time or processing-time and on ranges specified as time interval or row-count. The supported over window definitions are exposed as methods on Over (and other classes) and are listed below:

Partition By

Optional

Defines a partitioning of the input on one or more attributes. Each partition is individually sorted and aggregate functions are applied to each partition separately.

Note: In streaming environments, over window aggregates can only be computed in parallel if the window includes a partition by clause. Without partitionBy(…) the stream is processed by a single, non-parallel task.

Order By

Required

Defines the order of rows within each partition and thereby the order in which the aggregate functions are applied to rows.

Note: For streaming queries this must be a declared event-time or processing-time time attribute. Currently, only a single sort attribute is supported.

Preceding

Optional

Defines the interval of rows that are included in the window and precede the current row. The interval can either be specified as time or row-count interval.

Bounded over windows are specified with the size of the interval, e.g., 10.minutes for a time interval or 10.rows for a row-count interval.

Unbounded over windows are specified using a constant, i.e., UNBOUNDED_RANGE for a time interval or UNBOUNDED_ROW for a row-count interval. Unbounded over windows start with the first row of a partition.

If the preceding clause is omitted, UNBOUNDED_RANGE and CURRENT_RANGE are used as the default preceding and following for the window.

Following

Optional

Defines the window interval of rows that are included in the window and follow the current row. The interval must be specified in the same unit as the preceding interval (time or row-count).

At the moment, over windows with rows following the current row are not supported. Instead you can specify one of two constants:

  • CURRENT_ROW sets the upper bound of the window to the current row.
  • CURRENT_RANGE sets the upper bound of the window to sort key of the current row, i.e., all rows with the same sort key as the current row are included in the window.

If the following clause is omitted, the upper bound of a time interval window is defined as CURRENT_RANGE and the upper bound of a row-count interval window is defined as CURRENT_ROW.

As

Required

Assigns an alias to the over window. The alias is used to reference the over window in the following select() clause.

Note: Currently, all aggregation functions in the same select() call must be computed of the same over window.

Unbounded Over Windows

Java

  1. // Unbounded Event-time over window (assuming an event-time attribute "rowtime")
  2. .window(Over.partitionBy($("a")).orderBy($("rowtime")).preceding(UNBOUNDED_RANGE).as("w"));
  3. // Unbounded Processing-time over window (assuming a processing-time attribute "proctime")
  4. .window(Over.partitionBy($("a")).orderBy("proctime").preceding(UNBOUNDED_RANGE).as("w"));
  5. // Unbounded Event-time Row-count over window (assuming an event-time attribute "rowtime")
  6. .window(Over.partitionBy($("a")).orderBy($("rowtime")).preceding(UNBOUNDED_ROW).as("w"));
  7. // Unbounded Processing-time Row-count over window (assuming a processing-time attribute "proctime")
  8. .window(Over.partitionBy($("a")).orderBy($("proctime")).preceding(UNBOUNDED_ROW).as("w"));

Scala

  1. // Unbounded Event-time over window (assuming an event-time attribute "rowtime")
  2. .window(Over partitionBy $"a" orderBy $"rowtime" preceding UNBOUNDED_RANGE as "w")
  3. // Unbounded Processing-time over window (assuming a processing-time attribute "proctime")
  4. .window(Over partitionBy $"a" orderBy $"proctime" preceding UNBOUNDED_RANGE as "w")
  5. // Unbounded Event-time Row-count over window (assuming an event-time attribute "rowtime")
  6. .window(Over partitionBy $"a" orderBy $"rowtime" preceding UNBOUNDED_ROW as "w")
  7. // Unbounded Processing-time Row-count over window (assuming a processing-time attribute "proctime")
  8. .window(Over partitionBy $"a" orderBy $"proctime" preceding UNBOUNDED_ROW as "w")

Python

  1. # Unbounded Event-time over window (assuming an event-time attribute "rowtime")
  2. .over_window(Over.partition_by(col('a')).order_by(col('rowtime')).preceding(UNBOUNDED_RANGE).alias("w"))
  3. # Unbounded Processing-time over window (assuming a processing-time attribute "proctime")
  4. .over_window(Over.partition_by(col('a')).order_by(col('proctime')).preceding(UNBOUNDED_RANGE).alias("w"))
  5. # Unbounded Event-time Row-count over window (assuming an event-time attribute "rowtime")
  6. .over_window(Over.partition_by(col('a')).order_by(col('rowtime')).preceding(UNBOUNDED_ROW).alias("w"))
  7. # Unbounded Processing-time Row-count over window (assuming a processing-time attribute "proctime")
  8. .over_window(Over.partition_by(col('a')).order_by(col('proctime')).preceding(UNBOUNDED_ROW).alias("w"))

Bounded Over Windows

Java

  1. // Bounded Event-time over window (assuming an event-time attribute "rowtime")
  2. .window(Over.partitionBy($("a")).orderBy($("rowtime")).preceding(lit(1).minutes()).as("w"));
  3. // Bounded Processing-time over window (assuming a processing-time attribute "proctime")
  4. .window(Over.partitionBy($("a")).orderBy($("proctime")).preceding(lit(1).minutes()).as("w"));
  5. // Bounded Event-time Row-count over window (assuming an event-time attribute "rowtime")
  6. .window(Over.partitionBy($("a")).orderBy($("rowtime")).preceding(rowInterval(10)).as("w"));
  7. // Bounded Processing-time Row-count over window (assuming a processing-time attribute "proctime")
  8. .window(Over.partitionBy($("a")).orderBy($("proctime")).preceding(rowInterval(10)).as("w"));

Scala

  1. // Bounded Event-time over window (assuming an event-time attribute "rowtime")
  2. .window(Over partitionBy $"a" orderBy $"rowtime" preceding 1.minutes as "w")
  3. // Bounded Processing-time over window (assuming a processing-time attribute "proctime")
  4. .window(Over partitionBy $"a" orderBy $"proctime" preceding 1.minutes as "w")
  5. // Bounded Event-time Row-count over window (assuming an event-time attribute "rowtime")
  6. .window(Over partitionBy $"a" orderBy $"rowtime" preceding 10.rows as "w")
  7. // Bounded Processing-time Row-count over window (assuming a processing-time attribute "proctime")
  8. .window(Over partitionBy $"a" orderBy $"proctime" preceding 10.rows as "w")

Python

  1. # Bounded Event-time over window (assuming an event-time attribute "rowtime")
  2. .over_window(Over.partition_by(col('a')).order_by(col('rowtime')).preceding(lit(1).minutes).alias("w"))
  3. # Bounded Processing-time over window (assuming a processing-time attribute "proctime")
  4. .over_window(Over.partition_by(col('a')).order_by(col('proctime')).preceding(lit(1).minutes).alias("w"))
  5. # Bounded Event-time Row-count over window (assuming an event-time attribute "rowtime")
  6. .over_window(Over.partition_by(col('a')).order_by(col('rowtime')).preceding(row_interval(10)).alias("w"))
  7. # Bounded Processing-time Row-count over window (assuming a processing-time attribute "proctime")
  8. .over_window(Over.partition_by(col('a')).order_by(col('proctime')).preceding(row_interval(10)).alias("w"))

Row-based Operations

The row-based operations generate outputs with multiple columns.

Map

Batch Streaming

Java

Performs a map operation with a user-defined scalar function or built-in scalar function. The output will be flattened if the output type is a composite type.

  1. public class MyMapFunction extends ScalarFunction {
  2. public Row eval(String a) {
  3. return Row.of(a, "pre-" + a);
  4. }
  5. @Override
  6. public TypeInformation<?> getResultType(Class<?>[] signature) {
  7. return Types.ROW(Types.STRING(), Types.STRING());
  8. }
  9. }
  10. ScalarFunction func = new MyMapFunction();
  11. tableEnv.registerFunction("func", func);
  12. Table table = input
  13. .map(call("func", $("c")).as("a", "b"));

Scala

Performs a map operation with a user-defined scalar function or built-in scalar function. The output will be flattened if the output type is a composite type.

  1. class MyMapFunction extends ScalarFunction {
  2. def eval(a: String): Row = {
  3. Row.of(a, "pre-" + a)
  4. }
  5. override def getResultType(signature: Array[Class[_]]): TypeInformation[_] =
  6. Types.ROW(Types.STRING, Types.STRING)
  7. }
  8. val func = new MyMapFunction()
  9. val table = input
  10. .map(func($"c")).as("a", "b")

Python

Performs a map operation with a python general scalar function or vectorized scalar function. The output will be flattened if the output type is a composite type.

  1. from pyflink.common import Row
  2. from pyflink.table import DataTypes
  3. from pyflink.table.udf import udf
  4. def map_function(a: Row) -> Row:
  5. return Row(a[0] + 1, a[1] * a[1])
  6. # map operation with a python general scalar function
  7. func = udf(map_function, result_type=DataTypes.ROW(
  8. [DataTypes.FIELD("a", DataTypes.BIGINT()),
  9. DataTypes.FIELD("b", DataTypes.BIGINT())]))
  10. table = input.map(func).alias('a', 'b')
  11. # map operation with a python vectorized scalar function
  12. pandas_func = udf(lambda x: x * 2, result_type=DataTypes.ROW(
  13. [DataTypes.FIELD("a", DataTypes.BIGINT()),
  14. DataTypes.FIELD("b", DataTypes.BIGINT()))]),
  15. func_type='pandas')
  16. table = input.map(pandas_func).alias('a', 'b')

FlatMap

Batch Streaming

Java

Performs a flatMap operation with a table function.

  1. public class MyFlatMapFunction extends TableFunction<Row> {
  2. public void eval(String str) {
  3. if (str.contains("#")) {
  4. String[] array = str.split("#");
  5. for (int i = 0; i < array.length; ++i) {
  6. collect(Row.of(array[i], array[i].length()));
  7. }
  8. }
  9. }
  10. @Override
  11. public TypeInformation<Row> getResultType() {
  12. return Types.ROW(Types.STRING(), Types.INT());
  13. }
  14. }
  15. TableFunction func = new MyFlatMapFunction();
  16. tableEnv.registerFunction("func", func);
  17. Table table = input
  18. .flatMap(call("func", $("c")).as("a", "b"));

Scala

Performs a flatMap operation with a python table function.

  1. class MyFlatMapFunction extends TableFunction[Row] {
  2. def eval(str: String): Unit = {
  3. if (str.contains("#")) {
  4. str.split("#").foreach({ s =>
  5. val row = new Row(2)
  6. row.setField(0, s)
  7. row.setField(1, s.length)
  8. collect(row)
  9. })
  10. }
  11. }
  12. override def getResultType: TypeInformation[Row] = {
  13. Types.ROW(Types.STRING, Types.INT)
  14. }
  15. }
  16. val func = new MyFlatMapFunction
  17. val table = input
  18. .flatMap(func($"c")).as("a", "b")

Python

Performs a flat_map operation with a python table function.

  1. from pyflink.table.udf import udtf
  2. from pyflink.table import DataTypes
  3. from pyflink.common import Row
  4. @udtf(result_types=[DataTypes.INT(), DataTypes.STRING()])
  5. def split(x: Row) -> Row:
  6. for s in x[1].split(","):
  7. yield x[0], s
  8. input.flat_map(split)

Aggregate

Batch Streaming Result

Java

Performs an aggregate operation with an aggregate function. You have to close the “aggregate” with a select statement and the select statement does not support aggregate functions. The output of aggregate will be flattened if the output type is a composite type.

  1. public class MyMinMaxAcc {
  2. public int min = 0;
  3. public int max = 0;
  4. }
  5. public class MyMinMax extends AggregateFunction<Row, MyMinMaxAcc> {
  6. public void accumulate(MyMinMaxAcc acc, int value) {
  7. if (value < acc.min) {
  8. acc.min = value;
  9. }
  10. if (value > acc.max) {
  11. acc.max = value;
  12. }
  13. }
  14. @Override
  15. public MyMinMaxAcc createAccumulator() {
  16. return new MyMinMaxAcc();
  17. }
  18. public void resetAccumulator(MyMinMaxAcc acc) {
  19. acc.min = 0;
  20. acc.max = 0;
  21. }
  22. @Override
  23. public Row getValue(MyMinMaxAcc acc) {
  24. return Row.of(acc.min, acc.max);
  25. }
  26. @Override
  27. public TypeInformation<Row> getResultType() {
  28. return new RowTypeInfo(Types.INT, Types.INT);
  29. }
  30. }
  31. AggregateFunction myAggFunc = new MyMinMax();
  32. tableEnv.registerFunction("myAggFunc", myAggFunc);
  33. Table table = input
  34. .groupBy($("key"))
  35. .aggregate(call("myAggFunc", $("a")).as("x", "y"))
  36. .select($("key"), $("x"), $("y"));

Scala

Performs an aggregate operation with an aggregate function. You have to close the “aggregate” with a select statement and the select statement does not support aggregate functions. The output of aggregate will be flattened if the output type is a composite type.

  1. case class MyMinMaxAcc(var min: Int, var max: Int)
  2. class MyMinMax extends AggregateFunction[Row, MyMinMaxAcc] {
  3. def accumulate(acc: MyMinMaxAcc, value: Int): Unit = {
  4. if (value < acc.min) {
  5. acc.min = value
  6. }
  7. if (value > acc.max) {
  8. acc.max = value
  9. }
  10. }
  11. override def createAccumulator(): MyMinMaxAcc = MyMinMaxAcc(0, 0)
  12. def resetAccumulator(acc: MyMinMaxAcc): Unit = {
  13. acc.min = 0
  14. acc.max = 0
  15. }
  16. override def getValue(acc: MyMinMaxAcc): Row = {
  17. Row.of(Integer.valueOf(acc.min), Integer.valueOf(acc.max))
  18. }
  19. override def getResultType: TypeInformation[Row] = {
  20. new RowTypeInfo(Types.INT, Types.INT)
  21. }
  22. }
  23. val myAggFunc = new MyMinMax
  24. val table = input
  25. .groupBy($"key")
  26. .aggregate(myAggFunc($"a") as ("x", "y"))
  27. .select($"key", $"x", $"y")

Python

Performs an aggregate operation with a python general aggregate function or vectorized aggregate function. You have to close the “aggregate” with a select statement and the select statement does not support aggregate functions. The output of aggregate will be flattened if the output type is a composite type.

  1. from pyflink.common import Row
  2. from pyflink.table import DataTypes
  3. from pyflink.table.udf import AggregateFunction, udaf
  4. class CountAndSumAggregateFunction(AggregateFunction):
  5. def get_value(self, accumulator):
  6. return Row(accumulator[0], accumulator[1])
  7. def create_accumulator(self):
  8. return Row(0, 0)
  9. def accumulate(self, accumulator, *args):
  10. accumulator[0] += 1
  11. accumulator[1] += args[0][1]
  12. def retract(self, accumulator, *args):
  13. accumulator[0] -= 1
  14. accumulator[1] -= args[0][1]
  15. def merge(self, accumulator, accumulators):
  16. for other_acc in accumulators:
  17. accumulator[0] += other_acc[0]
  18. accumulator[1] += other_acc[1]
  19. def get_accumulator_type(self):
  20. return DataTypes.ROW(
  21. [DataTypes.FIELD("a", DataTypes.BIGINT()),
  22. DataTypes.FIELD("b", DataTypes.BIGINT())])
  23. def get_result_type(self):
  24. return DataTypes.ROW(
  25. [DataTypes.FIELD("a", DataTypes.BIGINT()),
  26. DataTypes.FIELD("b", DataTypes.BIGINT())])
  27. function = CountAndSumAggregateFunction()
  28. agg = udaf(function,
  29. result_type=function.get_result_type(),
  30. accumulator_type=function.get_accumulator_type(),
  31. name=str(function.__class__.__name__))
  32. # aggregate with a python general aggregate function
  33. result = t.group_by(t.a) \
  34. .aggregate(agg.alias("c", "d")) \
  35. .select("a, c, d")
  36. # aggregate with a python vectorized aggregate function
  37. pandas_udaf = udaf(lambda pd: (pd.b.mean(), pd.b.max()),
  38. result_type=DataTypes.ROW(
  39. [DataTypes.FIELD("a", DataTypes.FLOAT()),
  40. DataTypes.FIELD("b", DataTypes.INT())]),
  41. func_type="pandas")
  42. t.aggregate(pandas_udaf.alias("a", "b")) \
  43. .select("a, b")

Group Window Aggregate

Batch Streaming

Groups and aggregates a table on a group window and possibly one or more grouping keys. You have to close the “aggregate” with a select statement. And the select statement does not support “*” or aggregate functions.

Java

  1. AggregateFunction myAggFunc = new MyMinMax();
  2. tableEnv.registerFunction("myAggFunc", myAggFunc);
  3. Table table = input
  4. .window(Tumble.over(lit(5).minutes())
  5. .on($("rowtime"))
  6. .as("w")) // define window
  7. .groupBy($("key"), $("w")) // group by key and window
  8. .aggregate(call("myAggFunc", $("a")).as("x", "y"))
  9. .select($("key"), $("x"), $("y"), $("w").start(), $("w").end()); // access window properties and aggregate results

Scala

  1. val myAggFunc = new MyMinMax
  2. val table = input
  3. .window(Tumble over 5.minutes on $"rowtime" as "w") // define window
  4. .groupBy($"key", $"w") // group by key and window
  5. .aggregate(myAggFunc($"a") as ("x", "y"))
  6. .select($"key", $"x", $"y", $"w".start, $"w".end) // access window properties and aggregate results

Python

  1. from pyflink.table import DataTypes
  2. from pyflink.table.udf import AggregateFunction, udaf
  3. pandas_udaf = udaf(lambda pd: (pd.b.mean(), pd.b.max()),
  4. result_type=DataTypes.ROW(
  5. [DataTypes.FIELD("a", DataTypes.FLOAT()),
  6. DataTypes.FIELD("b", DataTypes.INT())]),
  7. func_type="pandas")
  8. tumble_window = Tumble.over(expr.lit(1).hours) \
  9. .on(expr.col("rowtime")) \
  10. .alias("w")
  11. t.select(t.b, t.rowtime) \
  12. .window(tumble_window) \
  13. .group_by("w") \
  14. .aggregate(pandas_udaf.alias("d", "e")) \
  15. .select("w.rowtime, d, e")

FlatAggregate

Java

Similar to a GroupBy Aggregation. Groups the rows on the grouping keys with the following running table aggregation operator to aggregate rows group-wise. The difference from an AggregateFunction is that TableAggregateFunction may return 0 or more records for a group. You have to close the “flatAggregate” with a select statement. And the select statement does not support aggregate functions.

Instead of using emitValue to output results, you can also use the emitUpdateWithRetract method. Different from emitValue, emitUpdateWithRetract is used to emit values that have been updated. This method outputs data incrementally in retract mode, i.e., once there is an update, we have to retract old records before sending new updated ones. The emitUpdateWithRetract method will be used in preference to the emitValue method if both methods are defined in the table aggregate function, because the method is treated to be more efficient than emitValue as it can output values incrementally.

  1. /**
  2. * Accumulator for Top2.
  3. */
  4. public class Top2Accum {
  5. public Integer first;
  6. public Integer second;
  7. }
  8. /**
  9. * The top2 user-defined table aggregate function.
  10. */
  11. public class Top2 extends TableAggregateFunction<Tuple2<Integer, Integer>, Top2Accum> {
  12. @Override
  13. public Top2Accum createAccumulator() {
  14. Top2Accum acc = new Top2Accum();
  15. acc.first = Integer.MIN_VALUE;
  16. acc.second = Integer.MIN_VALUE;
  17. return acc;
  18. }
  19. public void accumulate(Top2Accum acc, Integer v) {
  20. if (v > acc.first) {
  21. acc.second = acc.first;
  22. acc.first = v;
  23. } else if (v > acc.second) {
  24. acc.second = v;
  25. }
  26. }
  27. public void merge(Top2Accum acc, java.lang.Iterable<Top2Accum> iterable) {
  28. for (Top2Accum otherAcc : iterable) {
  29. accumulate(acc, otherAcc.first);
  30. accumulate(acc, otherAcc.second);
  31. }
  32. }
  33. public void emitValue(Top2Accum acc, Collector<Tuple2<Integer, Integer>> out) {
  34. // emit the value and rank
  35. if (acc.first != Integer.MIN_VALUE) {
  36. out.collect(Tuple2.of(acc.first, 1));
  37. }
  38. if (acc.second != Integer.MIN_VALUE) {
  39. out.collect(Tuple2.of(acc.second, 2));
  40. }
  41. }
  42. }
  43. tEnv.registerFunction("top2", new Top2());
  44. Table orders = tableEnv.from("Orders");
  45. Table result = orders
  46. .groupBy($("key"))
  47. .flatAggregate(call("top2", $("a")).as("v", "rank"))
  48. .select($("key"), $("v"), $("rank");

Scala

Similar to a GroupBy Aggregation. Groups the rows on the grouping keys with the following running table aggregation operator to aggregate rows group-wise. The difference from an AggregateFunction is that TableAggregateFunction may return 0 or more records for a group. You have to close the “flatAggregate” with a select statement. And the select statement does not support aggregate functions.

Instead of using emitValue to output results, you can also use the emitUpdateWithRetract method. Different from emitValue, emitUpdateWithRetract is used to emit values that have been updated. This method outputs data incrementally in retract mode, i.e., once there is an update, we have to retract old records before sending new updated ones. The emitUpdateWithRetract method will be used in preference to the emitValue method if both methods are defined in the table aggregate function, because the method is treated to be more efficient than emitValue as it can output values incrementally.

  1. import java.lang.{Integer => JInteger}
  2. import org.apache.flink.table.api.Types
  3. import org.apache.flink.table.functions.TableAggregateFunction
  4. /**
  5. * Accumulator for top2.
  6. */
  7. class Top2Accum {
  8. var first: JInteger = _
  9. var second: JInteger = _
  10. }
  11. /**
  12. * The top2 user-defined table aggregate function.
  13. */
  14. class Top2 extends TableAggregateFunction[JTuple2[JInteger, JInteger], Top2Accum] {
  15. override def createAccumulator(): Top2Accum = {
  16. val acc = new Top2Accum
  17. acc.first = Int.MinValue
  18. acc.second = Int.MinValue
  19. acc
  20. }
  21. def accumulate(acc: Top2Accum, v: Int) {
  22. if (v > acc.first) {
  23. acc.second = acc.first
  24. acc.first = v
  25. } else if (v > acc.second) {
  26. acc.second = v
  27. }
  28. }
  29. def merge(acc: Top2Accum, its: JIterable[Top2Accum]): Unit = {
  30. val iter = its.iterator()
  31. while (iter.hasNext) {
  32. val top2 = iter.next()
  33. accumulate(acc, top2.first)
  34. accumulate(acc, top2.second)
  35. }
  36. }
  37. def emitValue(acc: Top2Accum, out: Collector[JTuple2[JInteger, JInteger]]): Unit = {
  38. // emit the value and rank
  39. if (acc.first != Int.MinValue) {
  40. out.collect(JTuple2.of(acc.first, 1))
  41. }
  42. if (acc.second != Int.MinValue) {
  43. out.collect(JTuple2.of(acc.second, 2))
  44. }
  45. }
  46. }
  47. val top2 = new Top2
  48. val orders: Table = tableEnv.from("Orders")
  49. val result = orders
  50. .groupBy($"key")
  51. .flatAggregate(top2($"a") as ($"v", $"rank"))
  52. .select($"key", $"v", $"rank")

Python

Performs a flat_aggregate operation with a python general Table Aggregate Function

Similar to a GroupBy Aggregation. Groups the rows on the grouping keys with the following running table aggregation operator to aggregate rows group-wise. The difference from an AggregateFunction is that TableAggregateFunction may return 0 or more records for a group. You have to close the “flat_aggregate” with a select statement. And the select statement does not support aggregate functions.

  1. from pyflink.common import Row
  2. from pyflink.table.udf import TableAggregateFunction, udtaf
  3. from pyflink.table import DataTypes
  4. class Top2(TableAggregateFunction):
  5. def emit_value(self, accumulator):
  6. yield Row(accumulator[0])
  7. yield Row(accumulator[1])
  8. def create_accumulator(self):
  9. return [None, None]
  10. def accumulate(self, accumulator, *args):
  11. if args[0][0] is not None:
  12. if accumulator[0] is None or args[0][0] > accumulator[0]:
  13. accumulator[1] = accumulator[0]
  14. accumulator[0] = args[0][0]
  15. elif accumulator[1] is None or args[0][0] > accumulator[1]:
  16. accumulator[1] = args[0][0]
  17. def retract(self, accumulator, *args):
  18. accumulator[0] = accumulator[0] - 1
  19. def merge(self, accumulator, accumulators):
  20. for other_acc in accumulators:
  21. self.accumulate(accumulator, other_acc[0])
  22. self.accumulate(accumulator, other_acc[1])
  23. def get_accumulator_type(self):
  24. return DataTypes.ARRAY(DataTypes.BIGINT())
  25. def get_result_type(self):
  26. return DataTypes.ROW(
  27. [DataTypes.FIELD("a", DataTypes.BIGINT())])
  28. mytop = udtaf(Top2())
  29. t = t_env.from_elements([(1, 'Hi', 'Hello'),
  30. (3, 'Hi', 'hi'),
  31. (5, 'Hi2', 'hi'),
  32. (7, 'Hi', 'Hello'),
  33. (2, 'Hi', 'Hello')], ['a', 'b', 'c'])
  34. result = t.select(t.a, t.c) \
  35. .group_by(t.c) \
  36. .flat_aggregate(mytop) \
  37. .select(t.a) \
  38. .flat_aggregate(mytop.alias("b"))

For streaming queries the required state to compute the query result might grow infinitely depending on the type of aggregation and the number of distinct grouping keys. Please provide a query configuration with valid retention interval to prevent excessive state size. See Query Configuration for details.

Data Types

Please see the dedicated page about data types.

Generic types and (nested) composite types (e.g., POJOs, tuples, rows, Scala case classes) can be fields of a row as well.

Fields of composite types with arbitrary nesting can be accessed with value access functions.

Generic types are treated as a black box and can be passed on or processed by user-defined functions.