Flink JDBC Driver

Flink JDBC Driver is a Java library for connecting and submitting SQL statements to SQL Gateway as the JDBC server.

Usage

Before using Flink JDBC driver, you need to start a SQL Gateway as the JDBC server and binds it with your Flink cluster. We now assume that you have a gateway started and connected to a running Flink cluster.

Dependency

All dependencies for JDBC driver have been packaged in flink-sql-jdbc-driver-bundle, you can download and add the jar file in your project.

Group IdArtifact IdJAR
org.apache.flinkflink-sql-jdbc-driver-bundleDownload

You can also add dependency of Flink JDBC driver in your maven or gradle project.

Maven Dependency

  1. <dependency>
  2. <groupId>org.apache.flink</groupId>
  3. <artifactId>flink-sql-jdbc-driver-bundle</artifactId>
  4. <version>{VERSION}</version>
  5. </dependency>

Use with a JDBC Tool

Use with Beeline

Beeline is the command line tool for accessing Apache Hive, but it also supports general JDBC drivers. To install Hive and beeline, see Hive documentation.

  1. Download flink-jdbc-driver-bundle-{VERSION}.jar from download page and add it to $HIVE_HOME/lib.
  2. Run beeline and connect to a Flink SQL gateway. As Flink SQL gateway currently ignores user names and passwords, just leave them empty.

    1. beeline> !connect jdbc:flink://localhost:8083
  3. Execute any statement you want.

Sample Commands

  1. Beeline version 3.1.3 by Apache Hive
  2. beeline> !connect jdbc:flink://localhost:8083
  3. Connecting to jdbc:flink://localhost:8083
  4. Enter username for jdbc:flink://localhost:8083:
  5. Enter password for jdbc:flink://localhost:8083:
  6. Connected to: Flink JDBC Driver (version 1.18-SNAPSHOT)
  7. Driver: org.apache.flink.table.jdbc.FlinkDriver (version 1.18-SNAPSHOT)
  8. 0: jdbc:flink://localhost:8083> CREATE TABLE T(
  9. . . . . . . . . . . . . . . . > a INT,
  10. . . . . . . . . . . . . . . . > b VARCHAR(10)
  11. . . . . . . . . . . . . . . . > ) WITH (
  12. . . . . . . . . . . . . . . . > 'connector' = 'filesystem',
  13. . . . . . . . . . . . . . . . > 'path' = 'file:///tmp/T.csv',
  14. . . . . . . . . . . . . . . . > 'format' = 'csv'
  15. . . . . . . . . . . . . . . . > );
  16. No rows affected (0.108 seconds)
  17. 0: jdbc:flink://localhost:8083> INSERT INTO T VALUES (1, 'Hi'), (2, 'Hello');
  18. +-----------------------------------+
  19. | job id |
  20. +-----------------------------------+
  21. | da22010cf1c962b377493fc4fc509527 |
  22. +-----------------------------------+
  23. 1 row selected (0.952 seconds)
  24. 0: jdbc:flink://localhost:8083> SELECT * FROM T;
  25. +----+--------+
  26. | a | b |
  27. +----+--------+
  28. | 1 | Hi |
  29. | 2 | Hello |
  30. +----+--------+
  31. 2 rows selected (1.142 seconds)
  32. 0: jdbc:flink://localhost:8083>

Use with SqlLine

SqlLine is a lightweight JDBC command line tool, it supports general JDBC drivers. You need to clone the codes from github and compile the project with mvn first.

  1. Download flink-jdbc-driver-bundle-{VERSION}.jar from download page and add it to target directory of SqlLine project. Notice that you need to copy slf4j-api-{slf4j.version}.jar to target which will be used by flink JDBC driver.
  2. Run SqlLine with command bin/sqlline and connect to a Flink SQL gateway. As Flink SQL gateway currently ignores user names and passwords, just leave them empty.

    1. sqlline> !connect jdbc:flink://localhost:8083
  3. Execute any statement you want.

Sample Commands

  1. sqlline version 1.12.0
  2. sqlline> !connect jdbc:flink://localhost:8083
  3. Enter username for jdbc:flink://localhost:8083:
  4. Enter password for jdbc:flink://localhost:8083:
  5. 0: jdbc:flink://localhost:8083> CREATE TABLE T(
  6. . . . . . . . . . . . . . . .)> a INT,
  7. . . . . . . . . . . . . . . .)> b VARCHAR(10)
  8. . . . . . . . . . . . . . . .)> ) WITH (
  9. . . . . . . . . . . . . . . .)> 'connector' = 'filesystem',
  10. . . . . . . . . . . . . . . .)> 'path' = 'file:///tmp/T.csv',
  11. . . . . . . . . . . . . . . .)> 'format' = 'csv'
  12. . . . . . . . . . . . . . . .)> );
  13. No rows affected (0.122 seconds)
  14. 0: jdbc:flink://localhost:8083> INSERT INTO T VALUES (1, 'Hi'), (2, 'Hello');
  15. +----------------------------------+
  16. | job id |
  17. +----------------------------------+
  18. | fbade1ab4450fc57ebd5269fdf60dcfd |
  19. +----------------------------------+
  20. 1 row selected (1.282 seconds)
  21. 0: jdbc:flink://localhost:8083> SELECT * FROM T;
  22. +---+-------+
  23. | a | b |
  24. +---+-------+
  25. | 1 | Hi |
  26. | 2 | Hello |
  27. +---+-------+
  28. 2 rows selected (1.955 seconds)
  29. 0: jdbc:flink://localhost:8083>

Use with Tableau

Tableau is an interactive data visualization software. It supports Other Database (JDBC) connection from version 2018.3. You’ll need Tableau with version >= 2018.3 to use Flink JDBC driver. For general usage of Other Database (JDBC) in Tableau, see Tableau documentation.

  1. Download flink-jdbc-driver-(VERSION).jar from the download page and add it to Tableau driver path.
    • Windows: C:\Program Files\Tableau\Drivers
    • Mac: ~/Library/Tableau/Drivers
    • Linux: /opt/tableau/tableau_driver/jdbc
  2. Select Other Database (JDBC) under Connect and fill in the url of Flink SQL gateway. Select SQL92 dialect and leave user name and password empty.
  3. Hit Login button and use Tableau as usual.

Use with other JDBC Tools

Any tool supporting JDBC API can be used with Flink JDBC driver and Flink SQL gateway. See the documentation of your desired tool on how to use a JDBC driver.

Use with Application

Use with Java

Flink JDBC driver is a library for accessing Flink clusters through the JDBC API. For the general usage of JDBC in Java, see JDBC tutorial.

  1. Add the following dependency in pom.xml of project or download flink-jdbc-driver-bundle-{VERSION}.jar and add it to your classpath.
  2. Connect to a Flink SQL gateway in your Java code with specific url.
  3. Execute any statement you want.

Sample.java

  1. public class Sample {
  2. public static void main(String[] args) throws Exception {
  3. try (Connection connection = DriverManager.getConnection("jdbc:flink://localhost:8083")) {
  4. try (Statement statement = connection.createStatement()) {
  5. statement.execute("CREATE TABLE T(\n" +
  6. " a INT,\n" +
  7. " b VARCHAR(10)\n" +
  8. ") WITH (\n" +
  9. " 'connector' = 'filesystem',\n" +
  10. " 'path' = 'file:///tmp/T.csv',\n" +
  11. " 'format' = 'csv'\n" +
  12. ")");
  13. statement.execute("INSERT INTO T VALUES (1, 'Hi'), (2, 'Hello')");
  14. try (ResultSet rs = statement.executeQuery("SELECT * FROM T")) {
  15. while (rs.next()) {
  16. System.out.println(rs.getInt(1) + ", " + rs.getString(2));
  17. }
  18. }
  19. }
  20. }
  21. }
  22. }

Output

  1. 1, Hi
  2. 2, Hello

Besides DriverManager, Flink JDBC driver supports DataSource and you can also create connection from it.

DataSource.java

  1. public class Sample {
  2. public static void main(String[] args) throws Exception {
  3. DataSource dataSource = new FlinkDataSource("jdbc:flink://localhost:8083", new Properties());
  4. try (Connection connection = dataSource.getConnection()) {
  5. try (Statement statement = connection.createStatement()) {
  6. statement.execute("CREATE TABLE T(\n" +
  7. " a INT,\n" +
  8. " b VARCHAR(10)\n" +
  9. ") WITH (\n" +
  10. " 'connector' = 'filesystem',\n" +
  11. " 'path' = 'file:///tmp/T.csv',\n" +
  12. " 'format' = 'csv'\n" +
  13. ")");
  14. statement.execute("INSERT INTO T VALUES (1, 'Hi'), (2, 'Hello')");
  15. try (ResultSet rs = statement.executeQuery("SELECT * FROM T")) {
  16. while (rs.next()) {
  17. System.out.println(rs.getInt(1) + ", " + rs.getString(2));
  18. }
  19. }
  20. }
  21. }
  22. }
  23. }

Use with Others

In addition to java, Flink JDBC driver can be used by any JVM language such as scala, kotlin and ect, you can add the dependency of Flink JDBC driver in your project and use it directly.

Most applications may use data access frameworks to access data, for example, JOOQ, MyBatis and Spring Data. You can config Flink JDBC driver in them to perform Flink queries on an exist Flink cluster, just like a regular database.