DELETE 语句

DELETE 语句可以用于根据条件来删除表中的数据

注意 目前, DELETE 语句仅支持批模式, 并且要求目标表实现了 SupportsRowLevelDelete 接口。 如果在一个没有实现该接口的表上执行 DELETE,则会抛异常。目前 Flink 内置的连接器还没有实现该接口。

执行删除语句

Java

DELETE 语句,可以使用 TableEnvironment 中的 executeSql() 方法执行。executeSql() 方法执行 DELETE 语句时会立即提交一个 Flink 作业,并且返回一个 TableResult 对象,通过该对象可以获取 JobClient 方便的操作提交的作业。

以下的例子展示了如何在 TableEnvironment 中执行一条 DELETE 语句。

Scala

DELETE 语句,可以使用 TableEnvironment 中的 executeSql() 方法执行。executeSql() 方法执行 DELETE 语句时会立即提交一个 Flink 作业,并且返回一个 TableResult 对象,通过该对象可以获取 JobClient 方便的操作提交的作业。

以下的例子展示了如何在 TableEnvironment 中执行一条 DELETE 语句。

Python

DELETE 语句,可以使用 TableEnvironment 中的 execute_sql() 方法执行。execute_sql() 方法执行 DELETE 语句时会立即提交一个 Flink 作业,并且返回一个 TableResult 对象,通过该对象可以获取 JobClient 方便的操作提交的作业。

以下的例子展示了如何在 TableEnvironment 中执行一条 DELETE 语句。

SQL CLI

可以在 SQL CLI 中执行 DELETE 语句

以下的例子展示了如何在 SQL CLI 中执行一条 DELETE 语句。

Java

  1. EnvironmentSettings settings = EnvironmentSettings.newInstance().inBatchMode().build();
  2. TableEnvironment tEnv = TableEnvironment.create(settings);
  3. // 注册一个 "Orders" 表
  4. tEnv.executeSql("CREATE TABLE Orders (`user` STRING, product STRING, amount INT) WITH (...)");
  5. // 插入原始数据
  6. tEnv.executeSql("INSERT INTO Orders VALUES ('Lili', 'Apple', 1), ('Jessica', 'Banana', 2), ('Mr.White', 'Chicken', 3)").await();
  7. tEnv.executeSql("SELECT * FROM Orders").print();
  8. // +--------------------------------+--------------------------------+-------------+
  9. // | user | product | amount |
  10. // +--------------------------------+--------------------------------+-------------+
  11. // | Lili | Apple | 1 |
  12. // | Jessica | Banana | 2 |
  13. // | Mr.White | Chicken | 3 |
  14. // +--------------------------------+--------------------------------+-------------+
  15. // 3 rows in set
  16. // 根据where条件删除
  17. tEnv.executeSql("DELETE FROM Orders WHERE `user` = 'Lili'").await();
  18. tEnv.executeSql("SELECT * FROM Orders").print();
  19. // +--------------------------------+--------------------------------+-------------+
  20. // | user | product | amount |
  21. // +--------------------------------+--------------------------------+-------------+
  22. // | Jessica | Banana | 2 |
  23. // | Mr.White | Chicken | 3 |
  24. // +--------------------------------+--------------------------------+-------------+
  25. // 2 rows in set
  26. // 全表删除
  27. tEnv.executeSql("DELETE FROM Orders").await();
  28. tEnv.executeSql("SELECT * FROM Orders").print();
  29. // Empty set

Scala

  1. val env = StreamExecutionEnvironment.getExecutionEnvironment()
  2. val settings = EnvironmentSettings.newInstance().inBatchMode().build()
  3. val tEnv = StreamTableEnvironment.create(env, settings)
  4. // 注册一个 "Orders" 表
  5. tEnv.executeSql("CREATE TABLE Orders (`user` STRING, product STRING, amount INT) WITH (...)");
  6. // 插入原始数据
  7. tEnv.executeSql("INSERT INTO Orders VALUES ('Lili', 'Apple', 1), ('Jessica', 'Banana', 2), ('Mr.White', 'Chicken', 3)").await();
  8. tEnv.executeSql("SELECT * FROM Orders").print();
  9. // +--------------------------------+--------------------------------+-------------+
  10. // | user | product | amount |
  11. // +--------------------------------+--------------------------------+-------------+
  12. // | Lili | Apple | 1 |
  13. // | Jessica | Banana | 2 |
  14. // | Mr.White | Chicken | 3 |
  15. // +--------------------------------+--------------------------------+-------------+
  16. // 3 rows in set
  17. // 根据where条件删除
  18. tEnv.executeSql("DELETE FROM Orders WHERE `user` = 'Lili'").await();
  19. tEnv.executeSql("SELECT * FROM Orders").print();
  20. // +--------------------------------+--------------------------------+-------------+
  21. // | user | product | amount |
  22. // +--------------------------------+--------------------------------+-------------+
  23. // | Jessica | Banana | 2 |
  24. // | Mr.White | Chicken | 3 |
  25. // +--------------------------------+--------------------------------+-------------+
  26. // 2 rows in set
  27. // 全表删除
  28. tEnv.executeSql("DELETE FROM Orders").await();
  29. tEnv.executeSql("SELECT * FROM Orders").print();
  30. // Empty set

Python

  1. env_settings = EnvironmentSettings.in_batch_mode()
  2. table_env = TableEnvironment.create(env_settings)
  3. # 注册一个 "Orders" 表
  4. table_env.executeSql("CREATE TABLE Orders (`user` STRING, product STRING, amount INT) WITH (...)");
  5. # 插入原始数据
  6. table_env.executeSql("INSERT INTO Orders VALUES ('Lili', 'Apple', 1), ('Jessica', 'Banana', 2), ('Mr.White', 'Chicken', 3)").wait();
  7. table_env.executeSql("SELECT * FROM Orders").print();
  8. # +--------------------------------+--------------------------------+-------------+
  9. # | user | product | amount |
  10. # +--------------------------------+--------------------------------+-------------+
  11. # | Lili | Apple | 1 |
  12. # | Jessica | Banana | 2 |
  13. # | Mr.White | Chicken | 3 |
  14. # +--------------------------------+--------------------------------+-------------+
  15. # 3 rows in set
  16. # 根据where条件删除
  17. table_env.executeSql("DELETE FROM Orders WHERE `user` = 'Lili'").wait();
  18. table_env.executeSql("SELECT * FROM Orders").print();
  19. # +--------------------------------+--------------------------------+-------------+
  20. # | user | product | amount |
  21. # +--------------------------------+--------------------------------+-------------+
  22. # | Jessica | Banana | 2 |
  23. # | Mr.White | Chicken | 3 |
  24. # +--------------------------------+--------------------------------+-------------+
  25. # 2 rows in set
  26. # 全表删除
  27. table_env.executeSql("DELETE FROM Orders").wait();
  28. table_env.executeSql("SELECT * FROM Orders").print();
  29. # Empty set

SQL CLI

  1. Flink SQL> SET 'execution.runtime-mode' = 'batch';
  2. [INFO] Session property has been set.
  3. Flink SQL> CREATE TABLE Orders (`user` STRING, product STRING, amount INT) with (...);
  4. [INFO] Execute statement succeed.
  5. Flink SQL> INSERT INTO Orders VALUES ('Lili', 'Apple', 1), ('Jessica', 'Banana', 1), ('Mr.White', 'Chicken', 3);
  6. [INFO] Submitting SQL update statement to the cluster...
  7. [INFO] SQL update statement has been successfully submitted to the cluster:
  8. Job ID: bd2c46a7b2769d5c559abd73ecde82e9
  9. Flink SQL> SELECT * FROM Orders;
  10. user product amount
  11. Lili Apple 1
  12. Jessica Banana 2
  13. Mr.White Chicken 3
  14. Flink SQL> DELETE FROM Orders WHERE `user` = 'Lili';
  15. user product amount
  16. Jessica Banana 2
  17. Mr.White Chicken 3

语法

  1. DELETE FROM [catalog_name.][db_name.]table_name [ WHERE condition ]