Intro to the Python Table API

This document is a short introduction to the PyFlink Table API, which is used to help novice users quickly understand the basic usage of PyFlink Table API. For advanced usage, please refer to other documents in this User Guide.

Common Structure of Python Table API Program

All Table API and SQL programs, both batch and streaming, follow the same pattern. The following code example shows the common structure of Table API and SQL programs.

  1. from pyflink.table import EnvironmentSettings, StreamTableEnvironment
  2. # 1. create a TableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_streaming_mode().use_blink_planner().build()
  4. table_env = StreamTableEnvironment.create(environment_settings=env_settings)
  5. # 2. create source Table
  6. table_env.execute_sql("""
  7. CREATE TABLE datagen (
  8. id INT,
  9. data STRING
  10. ) WITH (
  11. 'connector' = 'datagen',
  12. 'fields.id.kind' = 'sequence',
  13. 'fields.id.start' = '1',
  14. 'fields.id.end' = '10'
  15. )
  16. """)
  17. # 3. create sink Table
  18. table_env.execute_sql("""
  19. CREATE TABLE print (
  20. id INT,
  21. data STRING
  22. ) WITH (
  23. 'connector' = 'print'
  24. )
  25. """)
  26. # 4. query from source table and perform caculations
  27. # create a Table from a Table API query:
  28. source_table = table_env.from_path("datagen")
  29. # or create a Table from a SQL query:
  30. source_table = table_env.sql_query("SELECT * FROM datagen")
  31. result_table = source_table.select(source_table.id + 1, source_table.data)
  32. # 5. emit query result to sink table
  33. # emit a Table API result Table to a sink table:
  34. result_table.execute_insert("print").wait()
  35. # or emit results via SQL query:
  36. table_env.execute_sql("INSERT INTO print SELECT * FROM datagen").wait()

Create a TableEnvironment

The TableEnvironment is a central concept of the Table API and SQL integration. The following code example shows how to create a TableEnvironment:

  1. from pyflink.table import EnvironmentSettings, StreamTableEnvironment, BatchTableEnvironment
  2. # create a blink streaming TableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_streaming_mode().use_blink_planner().build()
  4. table_env = StreamTableEnvironment.create(environment_settings=env_settings)
  5. # create a blink batch TableEnvironment
  6. env_settings = EnvironmentSettings.new_instance().in_batch_mode().use_blink_planner().build()
  7. table_env = BatchTableEnvironment.create(environment_settings=env_settings)
  8. # create a flink streaming TableEnvironment
  9. env_settings = EnvironmentSettings.new_instance().in_streaming_mode().use_old_planner().build()
  10. table_env = StreamTableEnvironment.create(environment_settings=env_settings)
  11. # create a flink batch TableEnvironment
  12. env_settings = EnvironmentSettings.new_instance().in_batch_mode().use_old_planner().build()
  13. table_env = BatchTableEnvironment.create(environment_settings=env_settings)

For more details about the different ways to create a TableEnvironment, please refer to the TableEnvironment Documentation.

The TableEnvironment is responsible for:

Currently there are 2 planners available: flink planner and blink planner.

You should explicitly set which planner to use in the current program. We recommend using the blink planner as much as possible.

Create Tables

Table is a core component of the Python Table API. A Table is a logical representation of the intermediate result of a Table API Job.

A Table is always bound to a specific TableEnvironment. It is not possible to combine tables from different TableEnvironments in same query, e.g., to join or union them.

Create using a List Object

You can create a Table from a list object:

  1. # create a blink batch TableEnvironment
  2. from pyflink.table import EnvironmentSettings, BatchTableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_batch_mode().use_blink_planner().build()
  4. table_env = BatchTableEnvironment.create(environment_settings=env_settings)
  5. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')])
  6. table.to_pandas()

The result is:

  1. _1 _2
  2. 0 1 Hi
  3. 1 2 Hello

You can also create the Table with specified column names:

  1. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  2. table.to_pandas()

The result is:

  1. id data
  2. 0 1 Hi
  3. 1 2 Hello

By default the table schema is extracted from the data automatically.

If the automatically generated table schema isn’t satisfactory, you can specify it manually:

  1. table_without_schema = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  2. # by default the type of the "id" column is 64 bit int
  3. default_type = table_without_schema.to_pandas()["id"].dtype
  4. print('By default the type of the "id" column is %s.' % default_type)
  5. from pyflink.table import DataTypes
  6. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')],
  7. DataTypes.ROW([DataTypes.FIELD("id", DataTypes.TINYINT()),
  8. DataTypes.FIELD("data", DataTypes.STRING())]))
  9. # now the type of the "id" column is 8 bit int
  10. type = table.to_pandas()["id"].dtype
  11. print('Now the type of the "id" column is %s.' % type)

The result is:

  1. By default the type of the "id" column is int64.
  2. Now the type of the "id" column is int8.

Create using a Connector

You can create a Table using connector DDL:

  1. # create a blink stream TableEnvironment
  2. from pyflink.table import EnvironmentSettings, StreamTableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_streaming_mode().use_blink_planner().build()
  4. table_env = StreamTableEnvironment.create(environment_settings=env_settings)
  5. table_env.execute_sql("""
  6. CREATE TABLE random_source (
  7. id BIGINT,
  8. data TINYINT
  9. ) WITH (
  10. 'connector' = 'datagen',
  11. 'fields.id.kind'='sequence',
  12. 'fields.id.start'='1',
  13. 'fields.id.end'='3',
  14. 'fields.data.kind'='sequence',
  15. 'fields.data.start'='4',
  16. 'fields.data.end'='6'
  17. )
  18. """)
  19. table = table_env.from_path("random_source")
  20. table.to_pandas()

The result is:

  1. id data
  2. 0 2 5
  3. 1 1 4
  4. 2 3 6

Create using a Catalog

A TableEnvironment maintains a map of catalogs of tables which are created with an identifier.

The tables in a catalog may either be temporary, and tied to the lifecycle of a single Flink session, or permanent, and visible across multiple Flink sessions.

The tables and views created via SQL DDL, e.g. “create table …” and “create view …” are also stored in a catalog.

You can directly access the tables in a catalog via SQL.

If you want to use tables from a catalog with the Table API, you can use the “from_path” method to create the Table API objects:

  1. # prepare the catalog
  2. # register Table API tables in the catalog
  3. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  4. table_env.create_temporary_view('source_table', table)
  5. # create Table API table from catalog
  6. new_table = table_env.from_path('source_table')
  7. new_table.to_pandas()

The result is:

  1. id data
  2. 0 1 Hi
  3. 1 2 Hello

Write Queries

Write Table API Queries

The Table object offers many methods for applying relational operations. These methods return new Table objects representing the result of applying the relational operations on the input Table. These relational operations may be composed of multiple method calls, such as table.group_by(...).select(...).

The Table API documentation describes all Table API operations that are supported on streaming and batch tables.

The following example shows a simple Table API aggregation query:

  1. # using batch table environment to execute the queries
  2. from pyflink.table import EnvironmentSettings, BatchTableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_batch_mode().use_blink_planner().build()
  4. table_env = BatchTableEnvironment.create(environment_settings=env_settings)
  5. orders = table_env.from_elements([('Jack', 'FRANCE', 10), ('Rose', 'ENGLAND', 30), ('Jack', 'FRANCE', 20)],
  6. ['name', 'country', 'revenue'])
  7. # compute revenue for all customers from France
  8. revenue = orders \
  9. .select(orders.name, orders.country, orders.revenue) \
  10. .where(orders.country == 'FRANCE') \
  11. .group_by(orders.name) \
  12. .select(orders.name, orders.revenue.sum.alias('rev_sum'))
  13. revenue.to_pandas()

The result is:

  1. name rev_sum
  2. 0 Jack 30

Write SQL Queries

Flink’s SQL integration is based on Apache Calcite, which implements the SQL standard. SQL queries are specified as Strings.

The SQL documentation describes Flink’s SQL support for streaming and batch tables.

The following example shows a simple SQL aggregation query:

  1. # use a StreamTableEnvironment to execute the queries
  2. from pyflink.table import EnvironmentSettings, StreamTableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_streaming_mode().use_blink_planner().build()
  4. table_env = StreamTableEnvironment.create(environment_settings=env_settings)
  5. table_env.execute_sql("""
  6. CREATE TABLE random_source (
  7. id BIGINT,
  8. data TINYINT
  9. ) WITH (
  10. 'connector' = 'datagen',
  11. 'fields.id.kind'='sequence',
  12. 'fields.id.start'='1',
  13. 'fields.id.end'='8',
  14. 'fields.data.kind'='sequence',
  15. 'fields.data.start'='4',
  16. 'fields.data.end'='11'
  17. )
  18. """)
  19. table_env.execute_sql("""
  20. CREATE TABLE print_sink (
  21. id BIGINT,
  22. data_sum TINYINT
  23. ) WITH (
  24. 'connector' = 'print'
  25. )
  26. """)
  27. table_env.execute_sql("""
  28. INSERT INTO print_sink
  29. SELECT id, sum(data) as data_sum FROM
  30. (SELECT id / 2 as id, data FROM random_source)
  31. WHERE id > 1
  32. GROUP BY id
  33. """).wait()

The result is:

  1. 2> +I(4,11)
  2. 6> +I(2,8)
  3. 8> +I(3,10)
  4. 6> -U(2,8)
  5. 8> -U(3,10)
  6. 6> +U(2,15)
  7. 8> +U(3,19)

In fact, this shows the change logs received by the print sink. The output format of a change log is:

  1. {subtask id}> {message type}{string format of the value}

For example, “2> +I(4,11)” means this message comes from the 2nd subtask, and “+I” means it is an insert message. “(4, 11)” is the content of the message. In addition, “-U” means a retract record (i.e. update-before), which means this message should be deleted or retracted from the sink. “+U” means this is an update record (i.e. update-after), which means this message should be updated or inserted by the sink.

So, we get this result from the change logs above:

  1. (4, 11)
  2. (2, 15)
  3. (3, 19)

Mix the Table API and SQL

The Table objects used in Table API and the tables used in SQL can be freely converted to each other.

The following example shows how to use a Table object in SQL:

  1. # create a sink table to emit results
  2. table_env.execute_sql("""
  3. CREATE TABLE table_sink (
  4. id BIGINT,
  5. data VARCHAR
  6. ) WITH (
  7. 'connector' = 'print'
  8. )
  9. """)
  10. # convert the Table API table to a SQL view
  11. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  12. table_env.create_temporary_view('table_api_table', table)
  13. # emit the Table API table
  14. table_env.execute_sql("INSERT INTO table_sink SELECT * FROM table_api_table").wait()

The result is:

  1. 6> +I(1,Hi)
  2. 6> +I(2,Hello)

And the following example shows how to use SQL tables in the Table API:

  1. # create a sql source table
  2. table_env.execute_sql("""
  3. CREATE TABLE sql_source (
  4. id BIGINT,
  5. data TINYINT
  6. ) WITH (
  7. 'connector' = 'datagen',
  8. 'fields.id.kind'='sequence',
  9. 'fields.id.start'='1',
  10. 'fields.id.end'='4',
  11. 'fields.data.kind'='sequence',
  12. 'fields.data.start'='4',
  13. 'fields.data.end'='7'
  14. )
  15. """)
  16. # convert the sql table to Table API table
  17. table = table_env.from_path("sql_source")
  18. # or create the table from a sql query
  19. table = table_env.sql_query("SELECT * FROM sql_source")
  20. # emit the table
  21. table.to_pandas()

The result is:

  1. id data
  2. 0 2 5
  3. 1 1 4
  4. 2 4 7
  5. 3 3 6

Emit Results

Collect Results to Client

You can call the “to_pandas” method to convert a Table object to a pandas DataFrame:

  1. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  2. table.to_pandas()

The result is:

  1. id data
  2. 0 1 Hi
  3. 1 2 Hello

Note “to_pandas” will trigger the materialization of the table and collect table content to the memory of the client, it’s good practice to limit the number of rows collected via Table.limit. Note “to_pandas” is not supported by the flink planner, and not all data types can be emitted to pandas DataFrames.

Emit Results to One Sink Table

You can call the “execute_insert” method to emit the data in a Table object to a sink table:

  1. table_env.execute_sql("""
  2. CREATE TABLE sink_table (
  3. id BIGINT,
  4. data VARCHAR
  5. ) WITH (
  6. 'connector' = 'print'
  7. )
  8. """)
  9. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  10. table.execute_insert("sink_table").wait()

The result is:

  1. 6> +I(1,Hi)
  2. 6> +I(2,Hello)

This could also be done using SQL:

  1. table_env.create_temporary_view("table_source", table)
  2. table_env.execute_sql("INSERT INTO sink_table SELECT * FROM table_source").wait()

Emit Results to Multiple Sink Tables

You can use a StatementSet to emit the Tables to multiple sink tables in one job:

  1. # prepare source tables and sink tables
  2. table = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  3. table_env.create_temporary_view("simple_source", table)
  4. table_env.execute_sql("""
  5. CREATE TABLE first_sink_table (
  6. id BIGINT,
  7. data VARCHAR
  8. ) WITH (
  9. 'connector' = 'print'
  10. )
  11. """)
  12. table_env.execute_sql("""
  13. CREATE TABLE second_sink_table (
  14. id BIGINT,
  15. data VARCHAR
  16. ) WITH (
  17. 'connector' = 'print'
  18. )
  19. """)
  20. # create a statement set
  21. statement_set = table_env.create_statement_set()
  22. # emit the "table" object to the "first_sink_table"
  23. statement_set.add_insert("first_sink_table", table)
  24. # emit the "simple_source" to the "second_sink_table" via a insert sql query
  25. statement_set.add_insert_sql("INSERT INTO second_sink_table SELECT * FROM simple_source")
  26. # execute the statement set
  27. statement_set.execute().wait()

The result is:

  1. 7> +I(1,Hi)
  2. 7> +I(1,Hi)
  3. 7> +I(2,Hello)
  4. 7> +I(2,Hello)

Explain Tables

The Table API provides a mechanism to explain the logical and optimized query plans used to compute a Table. This is done through the Table.explain() or StatementSet.explain() methods. Table.explain()returns the plan of a Table. StatementSet.explain() returns the plan for multiple sinks. These methods return a string describing three things:

  1. the Abstract Syntax Tree of the relational query, i.e., the unoptimized logical query plan,
  2. the optimized logical query plan, and
  3. the physical execution plan.

TableEnvironment.explain_sql() and TableEnvironment.execute_sql() support executing an EXPLAIN statement to get the plans. Please refer to the EXPLAIN page for more details.

The following code shows how to use the Table.explain() method:

  1. # using a StreamTableEnvironment
  2. from pyflink.table import EnvironmentSettings, StreamTableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_streaming_mode().use_blink_planner().build()
  4. table_env = StreamTableEnvironment.create(environment_settings=env_settings)
  5. table1 = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  6. table2 = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  7. table = table1 \
  8. .where(table1.data.like('H%')) \
  9. .union_all(table2)
  10. print(table.explain())

The result is:

  1. == Abstract Syntax Tree ==
  2. LogicalUnion(all=[true])
  3. :- LogicalFilter(condition=[LIKE($1, _UTF-16LE'H%')])
  4. : +- LogicalTableScan(table=[[default_catalog, default_database, Unregistered_TableSource_201907291, source: [PythonInputFormatTableSource(id, data)]]])
  5. +- LogicalTableScan(table=[[default_catalog, default_database, Unregistered_TableSource_1709623525, source: [PythonInputFormatTableSource(id, data)]]])
  6. == Optimized Logical Plan ==
  7. Union(all=[true], union=[id, data])
  8. :- Calc(select=[id, data], where=[LIKE(data, _UTF-16LE'H%')])
  9. : +- LegacyTableSourceScan(table=[[default_catalog, default_database, Unregistered_TableSource_201907291, source: [PythonInputFormatTableSource(id, data)]]], fields=[id, data])
  10. +- LegacyTableSourceScan(table=[[default_catalog, default_database, Unregistered_TableSource_1709623525, source: [PythonInputFormatTableSource(id, data)]]], fields=[id, data])
  11. == Physical Execution Plan ==
  12. Stage 133 : Data Source
  13. content : Source: PythonInputFormatTableSource(id, data)
  14. Stage 134 : Operator
  15. content : SourceConversion(table=[default_catalog.default_database.Unregistered_TableSource_201907291, source: [PythonInputFormatTableSource(id, data)]], fields=[id, data])
  16. ship_strategy : FORWARD
  17. Stage 135 : Operator
  18. content : Calc(select=[id, data], where=[(data LIKE _UTF-16LE'H%')])
  19. ship_strategy : FORWARD
  20. Stage 136 : Data Source
  21. content : Source: PythonInputFormatTableSource(id, data)
  22. Stage 137 : Operator
  23. content : SourceConversion(table=[default_catalog.default_database.Unregistered_TableSource_1709623525, source: [PythonInputFormatTableSource(id, data)]], fields=[id, data])
  24. ship_strategy : FORWARD

The following code shows how to use the StatementSet.explain() method:

  1. # using a StreamTableEnvironment
  2. from pyflink.table import EnvironmentSettings, StreamTableEnvironment
  3. env_settings = EnvironmentSettings.new_instance().in_streaming_mode().use_blink_planner().build()
  4. table_env = StreamTableEnvironment.create(environment_settings=env_settings)
  5. table1 = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  6. table2 = table_env.from_elements([(1, 'Hi'), (2, 'Hello')], ['id', 'data'])
  7. table_env.execute_sql("""
  8. CREATE TABLE print_sink_table (
  9. id BIGINT,
  10. data VARCHAR
  11. ) WITH (
  12. 'connector' = 'print'
  13. )
  14. """)
  15. table_env.execute_sql("""
  16. CREATE TABLE black_hole_sink_table (
  17. id BIGINT,
  18. data VARCHAR
  19. ) WITH (
  20. 'connector' = 'blackhole'
  21. )
  22. """)
  23. statement_set = table_env.create_statement_set()
  24. statement_set.add_insert("print_sink_table", table1.where(table1.data.like('H%')))
  25. statement_set.add_insert("black_hole_sink_table", table2)
  26. print(statement_set.explain())

The result is:

  1. == Abstract Syntax Tree ==
  2. LogicalSink(table=[default_catalog.default_database.print_sink_table], fields=[id, data])
  3. +- LogicalFilter(condition=[LIKE($1, _UTF-16LE'H%')])
  4. +- LogicalTableScan(table=[[default_catalog, default_database, Unregistered_TableSource_541737614, source: [PythonInputFormatTableSource(id, data)]]])
  5. LogicalSink(table=[default_catalog.default_database.black_hole_sink_table], fields=[id, data])
  6. +- LogicalTableScan(table=[[default_catalog, default_database, Unregistered_TableSource_1437429083, source: [PythonInputFormatTableSource(id, data)]]])
  7. == Optimized Logical Plan ==
  8. Sink(table=[default_catalog.default_database.print_sink_table], fields=[id, data])
  9. +- Calc(select=[id, data], where=[LIKE(data, _UTF-16LE'H%')])
  10. +- LegacyTableSourceScan(table=[[default_catalog, default_database, Unregistered_TableSource_541737614, source: [PythonInputFormatTableSource(id, data)]]], fields=[id, data])
  11. Sink(table=[default_catalog.default_database.black_hole_sink_table], fields=[id, data])
  12. +- LegacyTableSourceScan(table=[[default_catalog, default_database, Unregistered_TableSource_1437429083, source: [PythonInputFormatTableSource(id, data)]]], fields=[id, data])
  13. == Physical Execution Plan ==
  14. Stage 139 : Data Source
  15. content : Source: PythonInputFormatTableSource(id, data)
  16. Stage 140 : Operator
  17. content : SourceConversion(table=[default_catalog.default_database.Unregistered_TableSource_541737614, source: [PythonInputFormatTableSource(id, data)]], fields=[id, data])
  18. ship_strategy : FORWARD
  19. Stage 141 : Operator
  20. content : Calc(select=[id, data], where=[(data LIKE _UTF-16LE'H%')])
  21. ship_strategy : FORWARD
  22. Stage 143 : Data Source
  23. content : Source: PythonInputFormatTableSource(id, data)
  24. Stage 144 : Operator
  25. content : SourceConversion(table=[default_catalog.default_database.Unregistered_TableSource_1437429083, source: [PythonInputFormatTableSource(id, data)]], fields=[id, data])
  26. ship_strategy : FORWARD
  27. Stage 142 : Data Sink
  28. content : Sink: Sink(table=[default_catalog.default_database.print_sink_table], fields=[id, data])
  29. ship_strategy : FORWARD
  30. Stage 145 : Data Sink
  31. content : Sink: Sink(table=[default_catalog.default_database.black_hole_sink_table], fields=[id, data])
  32. ship_strategy : FORWARD