MySQL CDC source connector

Support Those Engines

SeaTunnel Zeta
Flink

Description

The MySQL CDC connector allows for reading snapshot data and incremental data from MySQL database. This document describes how to set up the MySQL CDC connector to run SQL queries against MySQL databases.

Key features

Supported DataSource Info

Datasource Supported versions Driver Url Maven
MySQL
  • MySQL: 5.5, 5.6, 5.7, 8.0.x
  • RDS MySQL: 5.6, 5.7, 8.0.x
  • com.mysql.cj.jdbc.Driver jdbc:mysql://localhost:3306/test https://mvnrepository.com/artifact/mysql/mysql-connector-java/8.0.28

    Using Dependency

    Install Jdbc Driver

    1. You need to ensure that the jdbc driver jar package has been placed in directory ${SEATUNNEL_HOME}/plugins/.

    For SeaTunnel Zeta Engine

    1. You need to ensure that the jdbc driver jar package has been placed in directory ${SEATUNNEL_HOME}/lib/.

    Creating MySQL user

    You have to define a MySQL user with appropriate permissions on all databases that the Debezium MySQL connector monitors.

    1. Create the MySQL user:
    1. mysql> CREATE USER 'user'@'localhost' IDENTIFIED BY 'password';
    1. Grant the required permissions to the user:
    1. mysql> GRANT SELECT, RELOAD, SHOW DATABASES, REPLICATION SLAVE, REPLICATION CLIENT ON *.* TO 'user' IDENTIFIED BY 'password';
    1. Finalize the user’s permissions:
    1. mysql> FLUSH PRIVILEGES;

    Enabling the MySQL Binlog

    You must enable binary logging for MySQL replication. The binary logs record transaction updates for replication tools to propagate changes.

    1. Check whether the log-bin option is already on:
    1. mysql> show variables where variable_name in ('log_bin', 'binlog_format', 'binlog_row_image', 'gtid_mode', 'enforce_gtid_consistency');
    2. +--------------------------+----------------+
    3. | Variable_name | Value |
    4. +--------------------------+----------------+
    5. | binlog_format | ROW |
    6. | binlog_row_image | FULL |
    7. | enforce_gtid_consistency | ON |
    8. | gtid_mode | ON |
    9. | log_bin | ON |
    10. +--------------------------+----------------+
    11. 5 rows in set (0.00 sec)
    1. If inconsistent with the above results, configure your MySQL server configuration file($MYSQL_HOME/mysql.cnf) with the following properties, which are described in the table below:
    1. # Enable binary replication log and set the prefix, expiration, and log format.
    2. # The prefix is arbitrary, expiration can be short for integration tests but would
    3. # be longer on a production system. Row-level info is required for ingest to work.
    4. # Server ID is required, but this will vary on production systems
    5. server-id = 223344
    6. log_bin = mysql-bin
    7. expire_logs_days = 10
    8. binlog_format = row
    9. # mysql 5.6+ requires binlog_row_image to be set to FULL
    10. binlog_row_image = FULL
    11. # enable gtid mode
    12. # mysql 5.6+ requires gtid_mode to be set to ON
    13. gtid_mode = on
    14. enforce_gtid_consistency = on
    1. Restart MySQL Server
    1. /etc/inint.d/mysqld restart
    1. Confirm your changes by checking the binlog status once more:

    MySQL 5.5:

    1. mysql> show variables where variable_name in ('log_bin', 'binlog_format', 'binlog_row_image', 'gtid_mode', 'enforce_gtid_consistency');
    2. +--------------------------+----------------+
    3. | Variable_name | Value |
    4. +--------------------------+----------------+
    5. | binlog_format | ROW |
    6. | log_bin | ON |
    7. +--------------------------+----------------+
    8. 5 rows in set (0.00 sec)

    MySQL 5.6+:

    1. mysql> show variables where variable_name in ('log_bin', 'binlog_format', 'binlog_row_image', 'gtid_mode', 'enforce_gtid_consistency');
    2. +--------------------------+----------------+
    3. | Variable_name | Value |
    4. +--------------------------+----------------+
    5. | binlog_format | ROW |
    6. | binlog_row_image | FULL |
    7. | enforce_gtid_consistency | ON |
    8. | gtid_mode | ON |
    9. | log_bin | ON |
    10. +--------------------------+----------------+
    11. 5 rows in set (0.00 sec)

    Notes

    Setting up MySQL session timeouts

    When an initial consistent snapshot is made for large databases, your established connection could timeout while the tables are being read. You can prevent this behavior by configuring interactive_timeout and wait_timeout in your MySQL configuration file.

    • interactive_timeout: The number of seconds the server waits for activity on an interactive connection before closing it. See MySQL’s documentation for more details.
    • wait_timeout: The number of seconds the server waits for activity on a non-interactive connection before closing it. See MySQL’s documentation for more details.

    For more database settings see Debezium MySQL Connector

    Data Type Mapping

    Mysql Data Type SeaTunnel Data Type
    BIT(1)
    TINYINT(1)
    BOOLEAN
    TINYINT TINYINT
    TINYINT UNSIGNED
    SMALLINT
    SMALLINT
    SMALLINT UNSIGNED
    MEDIUMINT
    MEDIUMINT UNSIGNED
    INT
    INTEGER
    YEAR
    INT
    INT UNSIGNED
    INTEGER UNSIGNED
    BIGINT
    BIGINT
    BIGINT UNSIGNED DECIMAL(20,0)
    DECIMAL(p, s)
    DECIMAL(p, s) UNSIGNED
    NUMERIC(p, s)
    NUMERIC(p, s) UNSIGNED
    DECIMAL(p,s)
    FLOAT
    FLOAT UNSIGNED
    FLOAT
    DOUBLE
    DOUBLE UNSIGNED
    REAL
    REAL UNSIGNED
    DOUBLE
    CHAR
    VARCHAR
    TINYTEXT
    MEDIUMTEXT
    TEXT
    LONGTEXT
    ENUM
    JSON
    ENUM
    STRING
    DATE DATE
    TIME(s) TIME(s)
    DATETIME
    TIMESTAMP(s)
    TIMESTAMP(s)
    BINARY
    VARBINAR
    BIT(p)
    TINYBLOB
    MEDIUMBLOB
    BLOB
    LONGBLOB
    GEOMETRY
    BYTES

    Source Options

    Name Type Required Default Description
    base-url String Yes - The URL of the JDBC connection. Refer to a case: jdbc:mysql://localhost:3306:3306/test.
    username String Yes - Name of the database to use when connecting to the database server.
    password String Yes - Password to use when connecting to the database server.
    database-names List No - Database name of the database to monitor.
    table-names List Yes - Table name of the database to monitor. The table name needs to include the database name, for example: database_name.table_name
    table-names-config List No - Table config list. for example: [{“table”: “db1.schema1.table1”,”primaryKeys”:[“key1”]}]
    startup.mode Enum No INITIAL Optional startup mode for MySQL CDC consumer, valid enumerations are initial, earliest, latest and specific.
    initial: Synchronize historical data at startup, and then synchronize incremental data.
    earliest: Startup from the earliest offset possible.
    latest: Startup from the latest offset.
    specific: Startup from user-supplied specific offsets.
    startup.specific-offset.file String No - Start from the specified binlog file name. Note, This option is required when the startup.mode option used specific.
    startup.specific-offset.pos Long No - Start from the specified binlog file position. Note, This option is required when the startup.mode option used specific.
    stop.mode Enum No NEVER Optional stop mode for MySQL CDC consumer, valid enumerations are never, latest or specific.
    never: Real-time job don’t stop the source.
    latest: Stop from the latest offset.
    specific: Stop from user-supplied specific offset.
    stop.specific-offset.file String No - Stop from the specified binlog file name. Note, This option is required when the stop.mode option used specific.
    stop.specific-offset.pos Long No - Stop from the specified binlog file position. Note, This option is required when the stop.mode option used specific.
    snapshot.split.size Integer No 8096 The split size (number of rows) of table snapshot, captured tables are split into multiple splits when read the snapshot of table.
    snapshot.fetch.size Integer No 1024 The maximum fetch size for per poll when read table snapshot.
    server-id String No - A numeric ID or a numeric ID range of this database client, The numeric ID syntax is like 5400, the numeric ID range syntax is like ‘5400-5408’.
    Every ID must be unique across all currently-running database processes in the MySQL cluster. This connector joins the
    MySQL cluster as another server (with this unique ID) so it can read the binlog.
    By default, a random number is generated between 6500 and 2,148,492,146, though we recommend setting an explicit value.
    server-time-zone String No UTC The session time zone in database server. If not set, then ZoneId.systemDefault() is used to determine the server time zone.
    connect.timeout.ms Duration No 30000 The maximum time that the connector should wait after trying to connect to the database server before timing out.
    connect.max-retries Integer No 3 The max retry times that the connector should retry to build database server connection.
    connection.pool.size Integer No 20 The jdbc connection pool size.
    chunk-key.even-distribution.factor.upper-bound Double No 100 The upper bound of the chunk key distribution factor. This factor is used to determine whether the table data is evenly distributed. If the distribution factor is calculated to be less than or equal to this upper bound (i.e., (MAX(id) - MIN(id) + 1) / row count), the table chunks would be optimized for even distribution. Otherwise, if the distribution factor is greater, the table will be considered as unevenly distributed and the sampling-based sharding strategy will be used if the estimated shard count exceeds the value specified by sample-sharding.threshold. The default value is 100.0.
    chunk-key.even-distribution.factor.lower-bound Double No 0.05 The lower bound of the chunk key distribution factor. This factor is used to determine whether the table data is evenly distributed. If the distribution factor is calculated to be greater than or equal to this lower bound (i.e., (MAX(id) - MIN(id) + 1) / row count), the table chunks would be optimized for even distribution. Otherwise, if the distribution factor is less, the table will be considered as unevenly distributed and the sampling-based sharding strategy will be used if the estimated shard count exceeds the value specified by sample-sharding.threshold. The default value is 0.05.
    sample-sharding.threshold Integer No 1000 This configuration specifies the threshold of estimated shard count to trigger the sample sharding strategy. When the distribution factor is outside the bounds specified by chunk-key.even-distribution.factor.upper-bound and chunk-key.even-distribution.factor.lower-bound, and the estimated shard count (calculated as approximate row count / chunk size) exceeds this threshold, the sample sharding strategy will be used. This can help to handle large datasets more efficiently. The default value is 1000 shards.
    inverse-sampling.rate Integer No 1000 The inverse of the sampling rate used in the sample sharding strategy. For example, if this value is set to 1000, it means a 1/1000 sampling rate is applied during the sampling process. This option provides flexibility in controlling the granularity of the sampling, thus affecting the final number of shards. It’s especially useful when dealing with very large datasets where a lower sampling rate is preferred. The default value is 1000.
    exactly_once Boolean No false Enable exactly once semantic.
    format Enum No DEFAULT Optional output format for MySQL CDC, valid enumerations are DEFAULTCOMPATIBLE_DEBEZIUM_JSON.
    debezium Config No - Pass-through Debezium’s properties to Debezium Embedded Engine which is used to capture data changes from MySQL server.
    common-options no - Source plugin common parameters, please refer to Source Common Options for details

    Task Example

    Simple

    Support multi-table reading

    1. env {
    2. parallelism = 1
    3. job.mode = "STREAMING"
    4. checkpoint.interval = 10000
    5. }
    6. source {
    7. MySQL-CDC {
    8. base-url = "jdbc:mysql://localhost:3306/testdb"
    9. username = "root"
    10. password = "root@123"
    11. table-names = ["testdb.table1", "testdb.table2"]
    12. startup.mode = "initial"
    13. }
    14. }
    15. sink {
    16. Console {
    17. }
    18. }

    Support debezium-compatible format send to kafka

    Must be used with kafka connector sink, see compatible debezium format for details

    Support custom primary key for table

    1. env {
    2. parallelism = 1
    3. job.mode = "STREAMING"
    4. checkpoint.interval = 10000
    5. }
    6. source {
    7. MySQL-CDC {
    8. base-url = "jdbc:mysql://localhost:3306/testdb"
    9. username = "root"
    10. password = "root@123"
    11. table-names = ["testdb.table1", "testdb.table2"]
    12. table-names-config = [
    13. {
    14. table = "testdb.table2"
    15. primaryKeys = ["id"]
    16. }
    17. ]
    18. }
    19. }
    20. sink {
    21. Console {
    22. }
    23. }

    Changelog

    • Add MySQL CDC Source Connector

    next version