Debezium connector for Informix
Debezium’s Informix connector can capture row-level changes in the tables of a Informix database. For information about the Informix Database versions that are compatible with this connector, see the Debezium release overview.
This connector is strongly inspired by the Debezium implementation of IBM Db2, but uses the Informix Change Streams API for Java to capture transactional data. The Change Data Capture API captures data from databases that have full row logging enabled and captures transactions from the current logical log. The API processes all transactions sequentially.
The first time that a Debezium Informix connector connects to an Informix database, the connector reads a consistent snapshot of the tables for which the connector is configured to capture changes. By default, the connector captures changes from all non-system tables. To customize snapshot behavior, you can set configuration properties to specify the tables to include or exclude in the snapshot.
After the snapshot completes, the connector begins to emit change events for updates that are committed to tables that are in capture mode. By default, change events for a particular table go to a Kafka topic that has the same name as the table. Applications and services can then consume change event records from these topics.
The connector requires the use of the Informix Change Streams API for Java, which is packaged as part of the Informix JDBC installation and is available on Maven Central alongside the latest JDBC drivers. |
The Informix connector has been tested with Informix for Linux. It is expected that the connector would also work on other platforms such as Windows, and we’d love to get your feedback if you can confirm this to be the case.
Overview
The Debezium Informix connector is based on the Informix Change Data Capture API that enables Change Data Capture in Informix.
The database administrator must prepare the database and the database server for using the Change Data Capture API. See Preparing to use the Change Data Capture API.
After you place tables in capture mode, the connector can read change stream records for each table update and generate change events from them. The connector emits a change event record for each row-level insert, update, and delete operation. By default, change events records are sent to a Kafka topic that has the same name as the source table. Optionally, you can customize the name of the target topic. Client applications read the Kafka topics that correspond to the database tables of interest and can react to each row-level change event.
Typically, the database administrator puts a table into capture mode in the middle of the life of a table. This means that the connector does not have the complete history of all changes that have been made to the table. Therefore, when the Informix connector first connects to a particular Informix database, it starts by performing a consistent snapshot of each table that is in capture mode. After the connector completes the snapshot, the connector streams change events from the point at which the snapshot was made. In this way, the connector starts with a consistent view of the tables that are in capture mode, and does not drop any changes that were made while it was performing the snapshot.
Debezium connectors are tolerant of failures. As the connector reads and produces change events, it records the log sequence number (LSN) of the change stream record. The LSN is the position of the change event in the database log. If the connector stops for any reason, including communication failures, network problems, or crashes, upon restarting it continues reading the change stream where it left off. This behavior also applies to snapshots. That is, if the snapshot was not complete when the connector stopped, after a restart, the connector begins a new snapshot.
How the connector works
To optimally configure and run a Debezium Informix connector, it is helpful to understand how the connector performs snapshots, streams change events, determines Kafka topic names, and handles schema changes.
Snapshots
The Informix replication feature is not designed to store the complete history of database changes.
As a result, the Debezium Informix connector cannot retrieve the entire history of the database from the logs.
To enable the connector to establish a baseline for the current state of the database, the first time that the connector starts, it performs an initial consistent snapshot of the tables that are in capture mode.
For each change that the snapshot captures, the connector emits a read
event to the Kafka topic for the captured table.
The following workflow lists the steps that Debezium takes to create a snapshot.
These steps describe the process for a snapshot when the snapshot.mode
configuration property is set to its default value, which is initial
.
You can customize the way that the connector creates snapshots by changing the value of the snapshot.mode
property.
If you configure a different snapshot mode, the connector completes the snapshot by using a modified version of this workflow.
-
Establish a connection to the database.
-
Determine which tables are in capture mode and should be included in the snapshot. By default, the connector captures the data for all non-system tables. After the snapshot completes, the connector continues to stream data for the specified tables. If you want the connector to capture data only from specific tables, you can configure the connector to capture the data for a subset of tables or table elements by setting properties such as
table.include.list
ortable.exclude.list
. -
Obtain a lock on each of the tables in capture mode. This lock ensures that no schema changes can occur in those tables until the snapshot completes. The level of the lock is determined by the value of the
snapshot.isolation.mode
connector configuration property. -
Read the highest (most recent) LSN position in the server’s transaction log.
-
Capture the schema of all tables or all tables that are designated for capture. The connector persists schema information in its internal database schema history topic. The schema history provides information about the structure that is in effect when a change event occurs.
By default, the connector captures the schema of every table in the database that is in capture mode, including tables that are not configured for capture. If tables are not configured for capture, the initial snapshot captures only their structure; it does not capture any table data.
For more information about why snapshots persist schema information for tables that you did not include in the initial snapshot, see Understanding why initial snapshots capture the schema for all tables.
-
Release any locks obtained in Step 3. Other database clients can now write to any previously locked tables.
-
At the LSN position read in Step 4, the connector scans the tables that are designated for capture. During the scan, the connector completes the following tasks:
-
Confirms that the table was created before the snapshot began. If the table was created after the snapshot began, the connector skips the table. After the snapshot is complete, and the connector transitions to streaming, it emits change events for any tables that were created after the snapshot began.
-
Produces a
read
event for each row that is captured from a table. Allread
events contain the same LSN position, which is the LSN position that was obtained in step 4. -
Emits each
read
event to the Kafka topic for the source table. -
Releases data table locks, if applicable.
-
-
Record the successful completion of the snapshot in the connector offsets.
The resulting initial snapshot captures the current state of each row in the captured tables. From this baseline state, the connector captures subsequent changes as they occur.
After the snapshot process begins, if the process is interrupted due to connector failure, rebalancing, or other reasons, the process restarts after the connector restarts.
After the connector completes the initial snapshot, it continues streaming from the position that it read in Step 4 so that it does not miss any updates.
If the connector stops again for any reason, after it restarts, it resumes streaming changes from where it previously left off.
Setting | Description |
---|---|
|
The connector performs a snapshot every time that it starts. After the snapshot completes, the connector begins to stream event records for subsequent database changes. |
|
The connector performs a database snapshot as described in the default workflow for creating an initial snapshot. After the snapshot completes, the connector begins to stream event records for subsequent database changes. |
|
The connector performs a database snapshot. After the snapshot completes, the connector stops, and does not stream event records for subsequent database changes. |
|
Deprecated, see |
|
The connector captures the structure of all relevant tables, performing all the steps described in the default snapshot workflow, except that it does not create |
|
Set this option to restore a database schema history topic that is lost or corrupted.
After a restart, the connector runs a snapshot that rebuilds the topic from the source tables.
You can also set the property to periodically prune a database schema history topic that experiences unexpected growth. + WARNING: Do not use this mode to perform a snapshot if schema changes were committed to the database after the last connector shutdown. |
|
After the connector starts, it performs a snapshot only if it detects one of the following circumstances: * It cannot detect any topic offsets. * A previously recorded offset specifies a log position that is not available on the server. |
|
Set the snapshot mode to |
|
The |
Understanding why initial snapshots capture the schema history for all tables
The initial snapshot that a connector runs captures two types of information:
- Table data
-
Information about
INSERT
,UPDATE
, andDELETE
operations in tables that are named in the connector’stable.include.list
property. - Schema data
-
DDL statements that describe the structural changes that are applied to tables. Schema data is persisted to both the internal schema history topic, and to the connector’s schema change topic, if one is configured.
After you run an initial snapshot, you might notice that the snapshot captures schema information for tables that are not designated for capture. By default, initial snapshots are designed to capture schema information for every table that is present in the database, not only from tables that are designated for capture. Connectors require that the table’s schema is present in the schema history topic before they can capture a table. By enabling the initial snapshot to capture schema data for tables that are not part of the original capture set, Debezium prepares the connector to readily capture event data from these tables should that later become necessary. If the initial snapshot does not capture a table’s schema, you must add the schema to the history topic before the connector can capture data from the table.
In some cases, you might want to limit schema capture in the initial snapshot. This can be useful when you want to reduce the time required to complete a snapshot. Or when Debezium connects to the database instance through a user account that has access to multiple logical databases, but you want the connector to capture changes only from tables in a specific logic database.
-
Capturing data from tables not captured by the initial snapshot (no schema change)
-
Capturing data from tables not captured by the initial snapshot (schema change)
-
Setting the
schema.history.internal.store.only.captured.tables.ddl
property to specify the tables from which to capture schema information. -
Setting the
schema.history.internal.store.only.captured.databases.ddl
property to specify the logical databases from which to capture schema changes.
Capturing data from tables not captured by the initial snapshot (no schema change)
In some cases, you might want the connector to capture data from a table whose schema was not captured by the initial snapshot. Depending on the connector configuration, the initial snapshot might capture the table schema only for specific tables in the database. If the table schema is not present in the history topic, the connector fails to capture the table, and reports a missing schema error.
You might still be able to capture data from the table, but you must perform additional steps to add the table schema.
-
You want to capture data from a table with a schema that the connector did not capture during the initial snapshot.
-
No schema changes were applied to the table between the LSNs of the earliest and latest change table entry that the connector reads. For information about capturing data from a new table that has undergone structural changes, see Capturing data from tables not captured by the initial snapshot (schema change).
-
Stop the connector.
-
Remove the internal database schema history topic that is specified by the
schema.history.internal.kafka.topic property
. -
Clear the offsets in the configured Kafka Connect
offset.storage.topic
. For more information about how to remove offsets, see the Debezium community FAQ.Removing offsets should be performed only by advanced users who have experience in manipulating internal Kafka Connect data. This operation is potentially destructive, and should be performed only as a last resort.
-
Apply the following changes to the connector configuration:
-
(Optional) Set the value of
schema.history.internal.captured.tables.ddl
tofalse
. This setting causes the snapshot to capture the schema for all tables, and guarantees that, in the future, the connector can reconstruct the schema history for all tables.Snapshots that capture the schema for all tables require more time to complete.
-
Add the tables that you want the connector to capture to
table.include.list
. -
Set the
snapshot.mode
to one of the following values:initial
-
When you restart the connector, it takes a full snapshot of the database that captures the table data and table structures.
If you select this option, consider setting the value of theschema.history.internal.captured.tables.ddl
property tofalse
to enable the connector to capture the schema of all tables. schema_only
-
When you restart the connector, it takes a snapshot that captures only the table schema. Unlike a full data snapshot, this option does not capture any table data. Use this option if you want to restart the connector more quickly than with a full snapshot.
-
-
Restart the connector. The connector completes the type of snapshot specified by the
snapshot.mode
. -
(Optional) If the connector performed a
schema_only
snapshot, after the snapshot completes, initiate an incremental snapshot to capture data from the tables that you added. The connector runs the snapshot while it continues to stream real-time changes from the tables. Running an incremental snapshot captures the following data changes:-
For tables that the connector previously captured, the incremental snapsot captures changes that occur while the connector was down, that is, in the interval between the time that the connector was stopped, and the current restart.
-
For newly added tables, the incremental snapshot captures all existing table rows.
-
Capturing data from tables not captured by the initial snapshot (schema change)
If a schema change is applied to a table, records that are committed before the schema change have different structures than those that were committed after the change. When Debezium captures data from a table, it reads the schema history to ensure that it applies the correct schema to each event. If the schema is not present in the schema history topic, the connector is unable to capture the table, and an error results.
If you want to capture data from a table that was not captured by the initial snapshot, and the schema of the table was modified, you must add the schema to the history topic, if it is not already available. You can add the schema by running a new schema snapshot, or by running an initial snapshot for the table.
-
You want to capture data from a table with a schema that the connector did not capture during the initial snapshot.
-
A schema change was applied to the table so that the records to be captured do not have a uniform structure.
- Initial snapshot captured the schema for all tables (
store.only.captured.tables.ddl
was set tofalse
) -
-
Edit the
table.include.list
property to specify the tables that you want to capture. -
Restart the connector.
-
Initiate an incremental snapshot if you want to capture existing data from the newly added tables.
-
- Initial snapshot did not capture the schema for all tables (
store.only.captured.tables.ddl
was set totrue
) -
If the initial snapshot did not save the schema of the table that you want to capture, complete one of the following procedures:
- Procedure 1: Schema snapshot, followed by incremental snapshot
-
In this procedure, the connector first performs a schema snapshot. You can then initiate an incremental snapshot to enable the connector to synchronize data.
-
Stop the connector.
-
Remove the internal database schema history topic that is specified by the
schema.history.internal.kafka.topic property
. -
Clear the offsets in the configured Kafka Connect
offset.storage.topic
. For more information about how to remove offsets, see the Debezium community FAQ.Removing offsets should be performed only by advanced users who have experience in manipulating internal Kafka Connect data. This operation is potentially destructive, and should be performed only as a last resort.
-
Set values for properties in the connector configuration as described in the following steps:
-
Set the value of the
snapshot.mode
property toschema_only
. -
Edit the
table.include.list
to add the tables that you want to capture.
-
-
Restart the connector.
-
Wait for Debezium to capture the schema of the new and existing tables. Data changes that occurred any tables after the connector stopped are not captured.
-
To ensure that no data is lost, initiate an incremental snapshot.
-
- Procedure 2: Initial snapshot, followed by optional incremental snapshot
-
In this procedure the connector performs a full initial snapshot of the database. As with any initial snapshot, in a database with many large tables, running an initial snapshot can be a time-consuming operation. After the snapshot completes, you can optionally trigger an incremental snapshot to capture any changes that occur while the connector is off-line.
-
Stop the connector.
-
Remove the internal database schema history topic that is specified by the
schema.history.internal.kafka.topic property
. -
Clear the offsets in the configured Kafka Connect
offset.storage.topic
. For more information about how to remove offsets, see the Debezium community FAQ.Removing offsets should be performed only by advanced users who have experience in manipulating internal Kafka Connect data. This operation is potentially destructive, and should be performed only as a last resort.
-
Edit the
table.include.list
to add the tables that you want to capture. -
Set values for properties in the connector configuration as described in the following steps:
-
Set the value of the
snapshot.mode
property toinitial
. -
(Optional) Set
schema.history.internal.store.only.captured.tables.ddl
tofalse
.
-
-
Restart the connector. The connector takes a full database snapshot. After the snapshot completes, the connector transitions to streaming.
-
(Optional) To capture any data that changed while the connector was off-line, initiate an incremental snapshot.
-
Ad hoc snapshots
By default, a connector runs an initial snapshot operation only after it starts for the first time. Following this initial snapshot, under normal circumstances, the connector does not repeat the snapshot process. Any future change event data that the connector captures comes in through the streaming process only.
However, in some situations the data that the connector obtained during the initial snapshot might become stale, lost, or incomplete. To provide a mechanism for recapturing table data, Debezium includes an option to perform ad hoc snapshots. You might want to perform an ad hoc snapshot after any of the following changes occur in your Debezium environment:
-
The connector configuration is modified to capture a different set of tables.
-
Kafka topics are deleted and must be rebuilt.
-
Data corruption occurs due to a configuration error or some other problem.
You can re-run a snapshot for a table for which you previously captured a snapshot by initiating a so-called ad-hoc snapshot. Ad hoc snapshots require the use of signaling tables. You initiate an ad hoc snapshot by sending a signal request to the Debezium signaling table.
When you initiate an ad hoc snapshot of an existing table, the connector appends content to the topic that already exists for the table. If a previously existing topic was removed, Debezium can create a topic automatically if automatic topic creation is enabled.
Ad hoc snapshot signals specify the tables to include in the snapshot. The snapshot can capture the entire contents of the database, or capture only a subset of the tables in the database. Also, the snapshot can capture a subset of the contents of the table(s) in the database.
You specify the tables to capture by sending an execute-snapshot
message to the signaling table.
Set the type of the execute-snapshot
signal to incremental
or blocking
, and provide the names of the tables to include in the snapshot, as described in the following table:
Field | Default | Value | ||
---|---|---|---|---|
|
|
Specifies the type of snapshot that you want to run. |
||
|
N/A |
An array that contains regular expressions matching the fully-qualified names of the table to be snapshotted. |
||
|
N/A |
An optional string, which specifies a condition based on the column(s) of the table(s), to capture a
subset of the contents of the table(s).
|
||
|
N/A |
An optional array that specifies a set of additional conditions that the connector evaluates to determine the subset of records to include in a snapshot.
|
||
|
N/A |
An optional string that specifies the column name that the connector uses as the primary key of a table during the snapshot process. |
You initiate an ad hoc incremental snapshot by adding an entry with the execute-snapshot
signal type to the signaling table.
After the connector processes the message, it begins the snapshot operation.
The snapshot process reads the first and last primary key values and uses those values as the start and end point for each table.
Based on the number of entries in the table, and the configured chunk size, Debezium divides the table into chunks, and proceeds to snapshot each chunk, in succession, one at a time.
For more information, see Incremental snapshots.
You initiate an ad hoc blocking snapshot by adding an entry with the execute-snapshot
signal type to the signaling table.
After the connector processes the message, it begins the snapshot operation.
The connector temporarily stops streaming, and then initiates a snapshot of the specified table, following the same process that it uses during an initial snapshot.
After the snapshot completes, the connector resumes streaming.
For more information, see Blocking snapshots.
Incremental snapshots
To provide flexibility in managing snapshots, Debezium includes a supplementary snapshot mechanism, known as incremental snapshotting. Incremental snapshots rely on the Debezium mechanism for sending signals to a Debezium connector. Incremental snapshots are based on the DDD-3 design document.
In an incremental snapshot, instead of capturing the full state of a database all at once, as in an initial snapshot, Debezium captures each table in phases, in a series of configurable chunks. You can specify the tables that you want the snapshot to capture and the size of each chunk. The chunk size determines the number of rows that the snapshot collects during each fetch operation on the database. The default chunk size for incremental snapshots is 1024 rows.
As an incremental snapshot proceeds, Debezium uses watermarks to track its progress, maintaining a record of each table row that it captures. This phased approach to capturing data provides the following advantages over the standard initial snapshot process:
-
You can run incremental snapshots in parallel with streamed data capture, instead of postponing streaming until the snapshot completes. The connector continues to capture near real-time events from the change log throughout the snapshot process, and neither operation blocks the other.
-
If the progress of an incremental snapshot is interrupted, you can resume it without losing any data. After the process resumes, the snapshot begins at the point where it stopped, rather than recapturing the table from the beginning.
-
You can run an incremental snapshot on demand at any time, and repeat the process as needed to adapt to database updates. For example, you might re-run a snapshot after you modify the connector configuration to add a table to its
table.include.list
property.
When you run an incremental snapshot, Debezium sorts each table by primary key and then splits the table into chunks based on the configured chunk size.
Working chunk by chunk, it then captures each table row in a chunk.
For each row that it captures, the snapshot emits a READ
event.
That event represents the value of the row when the snapshot for the chunk began.
As a snapshot proceeds, it’s likely that other processes continue to access the database, potentially modifying table records.
To reflect such changes, INSERT
, UPDATE
, or DELETE
operations are committed to the transaction log as per usual.
Similarly, the ongoing Debezium streaming process continues to detect these change events and emits corresponding change event records to Kafka.
In some cases, the UPDATE
or DELETE
events that the streaming process emits are received out of sequence.
That is, the streaming process might emit an event that modifies a table row before the snapshot captures the chunk that contains the READ
event for that row.
When the snapshot eventually emits the corresponding READ
event for the row, its value is already superseded.
To ensure that incremental snapshot events that arrive out of sequence are processed in the correct logical order, Debezium employs a buffering scheme for resolving collisions.
Only after collisions between the snapshot events and the streamed events are resolved does Debezium emit an event record to Kafka.
To assist in resolving collisions between late-arriving READ
events and streamed events that modify the same table row, Debezium employs a so-called snapshot window.
The snapshot windows demarcates the interval during which an incremental snapshot captures data for a specified table chunk.
Before the snapshot window for a chunk opens, Debezium follows its usual behavior and emits events from the transaction log directly downstream to the target Kafka topic.
But from the moment that the snapshot for a particular chunk opens, until it closes, Debezium performs a de-duplication step to resolve collisions between events that have the same primary key..
For each data collection, the Debezium emits two types of events, and stores the records for them both in a single destination Kafka topic.
The snapshot records that it captures directly from a table are emitted as READ
operations.
Meanwhile, as users continue to update records in the data collection, and the transaction log is updated to reflect each commit, Debezium emits UPDATE
or DELETE
operations for each change.
As the snapshot window opens, and Debezium begins processing a snapshot chunk, it delivers snapshot records to a memory buffer.
During the snapshot windows, the primary keys of the READ
events in the buffer are compared to the primary keys of the incoming streamed events.
If no match is found, the streamed event record is sent directly to Kafka.
If Debezium detects a match, it discards the buffered READ
event, and writes the streamed record to the destination topic, because the streamed event logically supersede the static snapshot event.
After the snapshot window for the chunk closes, the buffer contains only READ
events for which no related transaction log events exist.
Debezium emits these remaining READ
events to the table’s Kafka topic.
The connector repeats the process for each snapshot chunk.
The Debezium connector for Informix does not support schema changes while an incremental snapshot is running. |
Triggering an incremental snapshot
Currently, the only way to initiate an incremental snapshot is to send an ad hoc snapshot signal to the signaling table on the source database.
You submit a signal to the signaling table as SQL INSERT
queries.
After Debezium detects the change in the signaling table, it reads the signal, and runs the requested snapshot operation.
The query that you submit specifies the tables to include in the snapshot, and, optionally, specifies the type of snapshot operation.
Currently supports the incremental
and blocking
types.
To specify the tables to include in the snapshot, provide a data-collections
array that lists the tables or an array of regular expressions used to match tables, for example,
{"data-collections": ["public.MyFirstTable", "public.MySecondTable"]}
The data-collections
array for an incremental snapshot signal has no default value.
If the data-collections
array is empty, Debezium detects that no action is required and does not perform a snapshot.
If the name of a table that you want to include in a snapshot contains a dot ( |
-
-
A signaling data collection exists on the source database.
-
The signaling data collection is specified in the
signal.data.collection
property.
-
-
Send a SQL query to add the ad hoc incremental snapshot request to the signaling table:
INSERT INTO <signalTable> (id, type, data) VALUES ('<id>', '<snapshotType>', '{"data-collections": ["<tableName>","<tableName>"],"type":"<snapshotType>","additional-conditions":[{"data-collection": "<tableName>", "filter": "<additional-condition>"}]}');
For example,
INSERT INTO myschema.debezium_signal (id, type, data) (1) values ('ad-hoc-1', (2) 'execute-snapshot', (3) '{"data-collections": ["schema1.table1", "schema2.table2"], (4) "type":"incremental", (5) "additional-conditions":[{"data-collection": "schema1.table1" ,"filter":"color=\'blue\'"}]}'); (6)
The values of the
id
,type
, anddata
parameters in the command correspond to the fields of the signaling table.The following table describes the parameters in the example:
Table 3. Descriptions of fields in a SQL command for sending an incremental snapshot signal to the signaling table Item Value Description 1
myschema.debezium_signal
Specifies the fully-qualified name of the signaling table on the source database.
2
ad-hoc-1
The
id
parameter specifies an arbitrary string that is assigned as theid
identifier for the signal request.
Use this string to identify logging messages to entries in the signaling table. Debezium does not use this string. Rather, during the snapshot, Debezium generates its ownid
string as a watermarking signal.3
execute-snapshot
The
type
parameter specifies the operation that the signal is intended to trigger.4
data-collections
A required component of the
data
field of a signal that specifies an array of table names or regular expressions to match table names to include in the snapshot.
The array lists regular expressions which match tables by their fully-qualified names, using the same format as you use to specify the name of the connector’s signaling table in thesignal.data.collection
configuration property.5
incremental
An optional
type
component of thedata
field of a signal that specifies the type of snapshot operation to run.
Currently supports theincremental
andblocking
types.
If you do not specify a value, the connector runs an incremental snapshot.6
additional-conditions
An optional array that specifies a set of additional conditions that the connector evaluates to determine the subset of records to include in a snapshot.
Each additional condition is an object withdata-collection
andfilter
properties. You can specify different filters for each data collection.
* Thedata-collection
property is the fully-qualified name of the data collection for which the filter will be applied. For more information about theadditional-conditions
parameter, see Ad hoc incremental snapshots withadditional-conditions
.
additional-conditions
If you want a snapshot to include only a subset of the content in a table, you can modify the signal request by appending an additional-conditions
parameter to the snapshot signal.
The SQL query for a typical snapshot takes the following form:
SELECT * FROM <tableName> ....
By adding an additional-conditions
parameter, you append a WHERE
condition to the SQL query, as in the following example:
SELECT * FROM <data-collection> WHERE <filter> ....
The following example shows a SQL query to send an ad hoc incremental snapshot request with an additional condition to the signaling table:
INSERT INTO <signalTable> (id, type, data) VALUES ('<id>', '<snapshotType>', '{"data-collections": ["<tableName>","<tableName>"],"type":"<snapshotType>","additional-conditions":[{"data-collection": "<tableName>", "filter": "<additional-condition>"}]}');
For example, suppose you have a products
table that contains the following columns:
-
id
(primary key) -
color
-
quantity
If you want an incremental snapshot of the products
table to include only the data items where color=blue
, you can use the following SQL statement to trigger the snapshot:
INSERT INTO myschema.debezium_signal (id, type, data) VALUES('ad-hoc-1', 'execute-snapshot', '{"data-collections": ["schema1.products"],"type":"incremental", "additional-conditions":[{"data-collection": "schema1.products", "filter": "color=blue"}]}');
The additional-conditions
parameter also enables you to pass conditions that are based on more than one column.
For example, using the products
table from the previous example, you can submit a query that triggers an incremental snapshot that includes the data of only those items for which color=blue
and quantity>10
:
INSERT INTO myschema.debezium_signal (id, type, data) VALUES('ad-hoc-1', 'execute-snapshot', '{"data-collections": ["schema1.products"],"type":"incremental", "additional-conditions":[{"data-collection": "schema1.products", "filter": "color=blue AND quantity>10"}]}');
The following example, shows the JSON for an incremental snapshot event that is captured by a connector.
{
"before":null,
"after": {
"pk":"1",
"value":"New data"
},
"source": {
...
"snapshot":"incremental" (1)
},
"op":"r", (2)
"ts_ms":"1620393591654",
"ts_us":"1620393591654547",
"ts_ns":"1620393591654547920",
"transaction":null
}
Item | Field name | Description |
---|---|---|
1 |
|
Specifies the type of snapshot operation to run. |
2 |
|
Specifies the event type. |
Using the Kafka signaling channel to trigger an incremental snapshot
You can send a message to the configured Kafka topic to request the connector to run an ad hoc incremental snapshot.
The key of the Kafka message must match the value of the topic.prefix
connector configuration option.
The value of the message is a JSON object with type
and data
fields.
The signal type is execute-snapshot
, and the data
field must have the following fields:
Field | Default | Value | ||
---|---|---|---|---|
|
|
The type of the snapshot to be executed.
Currently Debezium supports the |
||
|
N/A |
An array of comma-separated regular expressions that match the fully-qualified names of tables to include in the snapshot. |
||
|
N/A |
An optional string that specifies a condition that the connector evaluates to designate a subset of records to include in a snapshot.
|
||
|
N/A |
An optional array of additional conditions that specifies criteria that the connector evaluates to designate a subset of records to include in a snapshot. |
An example of the execute-snapshot Kafka message:
Key = `test_connector` Value = `{"type":"execute-snapshot","data": {"data-collections": ["schema1.table1", "schema1.table2"], "type": "INCREMENTAL"}}`
Debezium uses the additional-conditions
field to select a subset of a table’s content.
Typically, when Debezium runs a snapshot, it runs a SQL query such as:
SELECT * FROM <tableName> ….
When the snapshot request includes an additional-conditions
property, the data-collection
and filter
parameters of the property are appended to the SQL query, for example:
SELECT * FROM <data-collection> WHERE <filter> ….
For example, given a products
table with the columns id
(primary key), color
, and brand
, if you want a snapshot to include only content for which color='blue'
, when you request the snapshot, you could add the additional-conditions
property to filter the content:
Key = `test_connector` Value = `{"type":"execute-snapshot","data": {"data-collections": ["schema1.products"], "type": "INCREMENTAL", "additional-conditions": [{"data-collection": "schema1.products" ,"filter":"color='blue'"}]}}`
You can use the additional-conditions
property to pass conditions based on multiple columns.
For example, using the same products
table as in the previous example, if you want a snapshot to include only the content from the products
table for which color='blue'
, and brand='MyBrand'
, you could send the following request:
Key = `test_connector` Value = `{"type":"execute-snapshot","data": {"data-collections": ["schema1.products"], "type": "INCREMENTAL", "additional-conditions": [{"data-collection": "schema1.products" ,"filter":"color='blue' AND brand='MyBrand'"}]}}`
Stopping an incremental snapshot
You can also stop an incremental snapshot by sending a signal to the table on the source database.
You submit a stop snapshot signal to the table by sending a SQL INSERT
query.
After Debezium detects the change in the signaling table, it reads the signal, and stops the incremental snapshot operation if it’s in progress.
The query that you submit specifies the snapshot operation of incremental
, and, optionally, the tables of the current running snapshot to be removed.
-
-
A signaling data collection exists on the source database.
-
The signaling data collection is specified in the
signal.data.collection
property.
-
-
Send a SQL query to stop the ad hoc incremental snapshot to the signaling table:
INSERT INTO <signalTable> (id, type, data) values ('<id>', 'stop-snapshot', '{"data-collections": ["<tableName>","<tableName>"],"type":"incremental"}');
For example,
INSERT INTO myschema.debezium_signal (id, type, data) (1) values ('ad-hoc-1', (2) 'stop-snapshot', (3) '{"data-collections": ["schema1.table1", "schema2.table2"], (4) "type":"incremental"}'); (5)
The values of the
id
,type
, anddata
parameters in the signal command correspond to the fields of the signaling table.The following table describes the parameters in the example:
Table 5. Descriptions of fields in a SQL command for sending a stop incremental snapshot signal to the signaling table Item Value Description 1
myschema.debezium_signal
Specifies the fully-qualified name of the signaling table on the source database.
2
ad-hoc-1
The
id
parameter specifies an arbitrary string that is assigned as theid
identifier for the signal request.
Use this string to identify logging messages to entries in the signaling table. Debezium does not use this string.3
stop-snapshot
Specifies
type
parameter specifies the operation that the signal is intended to trigger.4
data-collections
An optional component of the
data
field of a signal that specifies an array of table names or regular expressions to match table names to remove from the snapshot.
The array lists regular expressions which match tables by their fully-qualified names, using the same format as you use to specify the name of the connector’s signaling table in thesignal.data.collection
configuration property. If this component of thedata
field is omitted, the signal stops the entire incremental snapshot that is in progress.5
incremental
A required component of the
data
field of a signal that specifies the type of snapshot operation that is to be stopped.
Currently, the only valid option isincremental
.
If you do not specify atype
value, the signal fails to stop the incremental snapshot.
Using the Kafka signaling channel to stop an incremental snapshot
You can send a signal message to the configured Kafka signaling topic to stop an ad hoc incremental snapshot.
The key of the Kafka message must match the value of the topic.prefix
connector configuration option.
The value of the message is a JSON object with type
and data
fields.
The signal type is stop-snapshot
, and the data
field must have the following fields:
Field | Default | Value |
---|---|---|
|
|
The type of the snapshot to be executed.
Currently Debezium supports only the |
|
N/A |
An optional array of comma-separated regular expressions that match the fully-qualified names of the tables to include in the snapshot. |
The following example shows a typical stop-snapshot
Kafka message:
Key = `test_connector` Value = `{"type":"stop-snapshot","data": {"data-collections": ["schema1.table1", "schema1.table2"], "type": "INCREMENTAL"}}`
Custom snapshotter SPI
For more advanced uses, you can fine-tune control of the snapshot by implementing one of the following interfaces:
io.debezium.snapshot.spi.Snapshotter
-
Controls whether the connector takes a snapshot.
io.debezium.snapshot.spi.SnapshotQuery
-
Controls how data is queried during a snapshot.
io.debezium.snapshot.spi.SnapshotLock
-
Controls whether the connector locks tables when taking a snapshot.
/**
* {@link Snapshotter} is used to determine the following details about the snapshot process:
* <p>
* - Whether a snapshot occurs. <br>
* - Whether streaming continues during the snapshot. <br>
* - Whether the snapshot includes schema (if supported). <br>
* - Whether to snapshot data or schema following an error.
* <p>
* Although Debezium provides many default snapshot modes,
* to provide more advanced functionality, such as partial snapshots,
* you can customize implementation of the interface.
* For more information, see the documentation.
*
*
*
*/
@Incubating
public interface Snapshotter extends Configurable {
/**
* @return the name of the snapshotter.
*
*
*/
String name();
/**
* @param offsetExists is {@code true} when the connector has an offset context (i.e. restarted)
* @param snapshotInProgress is {@code true} when the connector is started, but a snapshot is already in progress
*
* @return {@code true} if the snapshotter should take a data snapshot
*/
boolean shouldSnapshotData(boolean offsetExists, boolean snapshotInProgress);
/**
* @param offsetExists is {@code true} when the connector has an offset context (i.e. restarted)
* @param snapshotInProgress is {@code true} when the connector is started, but a snapshot is already in progress
*
* @return {@code true} if the snapshotter should take a schema snapshot
*/
boolean shouldSnapshotSchema(boolean offsetExists, boolean snapshotInProgress);
/**
* @return {@code true} if the snapshotter should stream after taking a snapshot
*/
boolean shouldStream();
/**
* @return {@code true} whether the schema can be recovered if database schema history is corrupted.
*/
boolean shouldSnapshotOnSchemaError();
/**
* @return {@code true} whether the snapshot should be re-executed when there is a gap in data stream.
*/
boolean shouldSnapshotOnDataError();
/**
*
* @return {@code true} if streaming should resume from the start of the snapshot
* transaction, or {@code false} for when a connector resumes and takes a snapshot,
* streaming should resume from where streaming previously left off.
*/
default boolean shouldStreamEventsStartingFromSnapshot() {
return true;
}
/**
* Lifecycle hook called after the snapshot phase is successful.
*/
default void snapshotCompleted() {
// no operation
}
/**
* Lifecycle hook called after the snapshot phase is aborted.
*/
default void snapshotAborted() {
// no operation
}
}
/**
* {@link SnapshotQuery} is used to determine the query used during a data snapshot
*
*
*/
public interface SnapshotQuery extends Configurable, Service {
/**
* @return the name of the snapshot lock.
*
*
*/
String name();
/**
* Generate a valid query string for the specified table, or an empty {@link Optional}
* to skip snapshotting this table (but that table will still be streamed from)
*
* @param tableId the table to generate a query for
* @param snapshotSelectColumns the columns to be used in the snapshot select based on the column
* include/exclude filters
* @return a valid query string, or none to skip snapshotting this table
*/
Optional<String> snapshotQuery(String tableId, List<String> snapshotSelectColumns);
}
/**
* {@link SnapshotLock} is used to determine the table lock mode used during schema snapshot
*
*
*/
public interface SnapshotLock extends Configurable, Service {
/**
* @return the name of the snapshot lock.
*
*
*/
String name();
/**
* Returns a SQL statement for locking the given table during snapshotting, if required by the specific snapshotter
* implementation.
*/
Optional<String> tableLockingStatement(Duration lockTimeout, String tableId);
}
Blocking snapshots
To provide more flexibility in managing snapshots, Debezium includes a supplementary ad hoc snapshot mechanism, known as a blocking snapshot. Blocking snapshots rely on the Debezium mechanism for sending signals to a Debezium connector.
A blocking snapshot behaves just like an initial snapshot, except that you can trigger it at run time.
You might want to run a blocking snapshot rather than use the standard initial snapshot process in the following situations:
-
You add a new table and you want to complete the snapshot while the connector is running.
-
You add a large table, and you want the snapshot to complete in less time than is possible with an incremental snapshot.
When you run a blocking snapshot, Debezium stops streaming, and then initiates a snapshot of the specified table, following the same process that it uses during an initial snapshot. After the snapshot completes, the streaming is resumed.
You can set the following properties in the data
component of a signal:
-
data-collections: to specify which tables must be snapshot
-
additional-conditions: You can specify different filters for different table.
-
The
data-collection
property is the fully-qualified name of the table for which the filter will be applied. -
The
filter
property will have the same value used in thesnapshot.select.statement.overrides
-
For example:
{"type": "blocking", "data-collections": ["schema1.table1", "schema1.table2"], "additional-conditions": [{"data-collection": "schema1.table1", "filter": "SELECT * FROM [schema1].[table1] WHERE column1 = 0 ORDER BY column2 DESC"}, {"data-collection": "schema1.table2", "filter": "SELECT * FROM [schema1].[table2] WHERE column2 > 0"}]}
A delay might exist between the time that you send the signal to trigger the snapshot, and the time when streaming stops and the snapshot starts. As a result of this delay, after the snapshot completes, the connector might emit some event records that duplicate records captured by the snapshot.
Change stream records
After a complete snapshot, when a Debezium Informix connector starts for the first time, the connector starts consuming change stream records for the source tables that are in capture mode. The connector does the following:
-
Reads available change records from the current LSN.
-
Groups records by transaction Id and orders them according to the change LSN for each record.
-
Processes records as transactions are committed.
-
Passes begin, commit and change LSNs as offsets to Kafka Connect.
-
Stores the highest commit LSN and the lowest, uncommitted begin LSN that the connector passed to Kafka Connect.
After a restart, the connector resumes emitting change events from the offset (begin, commit and change LSNs) where it left off. It does so by:
-
Reading change records that were created between the last stored, lowest uncommitted begin LSN and the current LSN.
-
Grouping records by transaction Id and ordering them according to the change LSN for each event.
-
Discarding already processed transactions (commit LSN lower than last stored commit LSN).
-
Discarding already processed records of the last incompletely processed transaction, if any (change LSN lower than last stored change LSN and commit LSN equal to last stored commit LSN).
-
Processes the remaining records of any incompletely processed transaction.
-
Continues processing records as transactions are committed.
Topic names
By default, the Informix connector writes change events for all of the INSERT
, UPDATE
, and DELETE
operations that occur in a table to a single Apache Kafka topic that is specific to that table.
The connector uses the following convention to name change event topics:
topicPrefix.schemaName.tableName
The following list provides definitions for the components of the default name:
- topicPrefix
-
The topic prefix as specified by the
topic.prefix
connector configuration property. - schemaName
-
The name of the schema in which the operation occurred.
- tableName
-
The name of the table in which the operation occurred.
For example, consider an Informix installation with a mydatabase
database that contains the following tables in the myschema
schema:
-
products
-
products_on_hand
-
customers
-
orders
The connector would emit events to the following Kafka topics:
-
mydatabase.myschema.products
-
mydatabase.myschema.products_on_hand
-
mydatabase.myschema.customers
-
mydatabase.myschema.orders
The connector applies similar naming conventions to label its internal database schema history topics, schema change topics, and transaction metadata topics.
If the default topic names do not meet your requirements, you can configure custom topic names. To configure custom topic names, you specify regular expressions in the logical topic routing SMT. For more information about using the logical topic routing SMT to customize topic naming, see Topic routing.
Schema history topic
When a database client queries a database, the client uses the database’s current schema. However, the database schema can be changed at any time, which means that the connector must be able to identify what the schema was at the time that each insert, update, or delete operation was recorded. Also, a connector cannot necessarily apply the current schema to every event. If an event is relatively old, it’s possible that it was recorded before the current schema was applied.
To ensure correct processing of events that occur after a schema change, the Debezium Informix connector stores a snapshot of the new schema based on the structures of the Informix change data tables, which mirror the structures of their associated data tables. The connector stores the table schema information, together with the LSN of operations the result in schema changes, in the database schema history Kafka topic. The connector uses the stored schema representation to produce change events that correctly mirror the structure of tables at the time of each insert, update, or delete operation.
When the connector restarts after either a crash or a graceful stop, it resumes reading entries in the Informix change data tables from the last position that it read. Based on the schema information that the connector reads from the database schema history topic, the connector applies the table structures that existed at the position where the connector restarts.
If you update the schema of an Informix table that is in capture mode, it’s important that you also update the schema of the corresponding change table. You must be a Informix database administrator with elevated privileges to update database schema. For more information about how to update Informix database schema in Debezium environments, see Schema history evolution.
The database schema history topic is for internal connector use only. Optionally, the connector can also emit schema change events to a different topic that is intended for consumer applications.
-
Default names for topics that receive Debezium event records.
Schema change topic
You can configure a Debezium Informix connector to produce schema change events that describe schema changes that are applied to tables in the database.
Debezium emits a message to the schema change topic after the following operations occur in the source database:
-
You enable Debezium to capture changes from a new table.
-
You disable capture for a table from which Debezium previously captured changes.
The connector writes schema change events to a Kafka schema change topic that has the name <topicPrefix>
where <topicPrefix>
is the topic prefix that is specified in the topic.prefix
connector configuration property.
The schema for the schema change event has the following elements:
name
-
The name of the schema change event message.
type
-
The type of the change event message.
version
-
The version of the schema. The version is an integer that is incremented each time the schema is changed.
fields
-
The fields that are included in the change event message.
The following example shows a typical schema in JSON format.
{
"schema": {
"type": "struct",
"fields": [
{
"type": "string",
"optional": false,
"field": "databaseName"
}
],
"optional": false,
"name": "io.debezium.connector.informix.SchemaChangeKey",
"version": 1
},
"payload": {
"databaseName": "inventory"
}
}
Messages that the connector sends to the schema change topic contain a payload that includes the following elements:
databaseName
-
The name of the database to which the statements are applied. The value of
databaseName
serves as the message key. pos
-
The position in the transaction log where the statements appear.
tableChanges
-
A structured representation of the entire table schema after the schema change. The
tableChanges
field contains an array that includes entries for each column of the table. Because the structured representation presents data in JSON or Avro format, consumers can easily read messages without first processing them through a DDL parser.
For a table that is in capture mode, the connector not only stores the history of schema changes in the schema change topic, but also in an internal database schema history topic. The internal database schema history topic is for connector use only and it is not intended for direct use by consuming applications. Ensure that applications that require notifications about schema changes consume that information only from the schema change topic. |
Never partition the database schema history topic. For the database schema history topic to function correctly, it must maintain a consistent, global order of the event records that the connector emits to it. To ensure that the topic is not split among partitions, set the partition count for the topic by using one of the following methods:
|
The format of messages that a connector emits to its schema change topic is in an incubating state and can change without notice. |
The following example shows a message in the schema change topic. The message contains a logical representation of the table schema.
{
"schema": {
...
},
"payload": {
"source": {
"version": "2.6.2.Final",
"connector": "informix",
"name": "informix",
"ts_ms": 1588252618953,
"snapshot": "true",
"db": "testdb",
"schema": "informix",
"table": "customers",
"commit_lsn": "0",
"change_lsn": "0",
"txId": null,
"begin_lsn": "0"
},
"ts_ms": 1588252618953, (1)
"databaseName": "testdb", (2)
"schemaName": "informix",
"ddl": null, (3)
"tableChanges": [ (4)
{
"type": "CREATE", (5)
"id": "\"testdb\".\"informix\".\"customers\"", (6)
"table": { (7)
"defaultCharsetName": null,
"primaryKeyColumnNames": [ (8)
"id"
],
"columns": [ (9)
{
"name": "id",
"jdbcType": 4,
"nativeType": null,
"typeName": "int identity",
"typeExpression": "int identity",
"charsetName": null,
"length": 10,
"scale": 0,
"position": 1,
"optional": false,
"autoIncremented": false,
"generated": false
},
{
"name": "first_name",
"jdbcType": 12,
"nativeType": null,
"typeName": "varchar",
"typeExpression": "varchar",
"charsetName": null,
"length": 255,
"scale": null,
"position": 2,
"optional": false,
"autoIncremented": false,
"generated": false
},
{
"name": "last_name",
"jdbcType": 12,
"nativeType": null,
"typeName": "varchar",
"typeExpression": "varchar",
"charsetName": null,
"length": 255,
"scale": null,
"position": 3,
"optional": false,
"autoIncremented": false,
"generated": false
},
{
"name": "email",
"jdbcType": 12,
"nativeType": null,
"typeName": "varchar",
"typeExpression": "varchar",
"charsetName": null,
"length": 255,
"scale": null,
"position": 4,
"optional": false,
"autoIncremented": false,
"generated": false
}
],
"attributes": [ (10)
{
"customAttribute": "attributeValue"
}
]
}
}
]
}
}
Item | Field name | Description | ||||||
---|---|---|---|---|---|---|---|---|
1 |
|
Optional field that displays the time at which the connector processed the event. The time is based on the system clock in the JVM running the Kafka Connect task. In the source object, |
||||||
2 |
|
Identifies the database and the schema that contain the change. |
||||||
3 |
|
Always |
||||||
4 |
|
An array of one or more items that contain the schema changes generated by a DDL command. |
||||||
5 |
|
Describes the type of change. The field contains one of the following values:
|
||||||
6 |
|
Full identifier of the table that was created, altered, or dropped. |
||||||
7 |
|
Represents table metadata after the applied change. |
||||||
8 |
|
List of columns that comprise the table’s primary key. |
||||||
9 |
|
Metadata for each column in the changed table. |
||||||
10 |
|
Custom attribute metadata for each table change. |
In messages that the connector sends to the schema change topic, the message key is the name of the database that contains the schema change.
In the following example, the payload
field contains the key:
{
"schema": {
"type": "struct",
"fields": [
{
"type": "string",
"optional": false,
"field": "databaseName"
}
],
"optional": false,
"name": "io.debezium.connector.informix.SchemaChangeKey",
"version": 1
},
"payload": {
"databaseName": "testdb"
}
}
Transaction metadata
Debezium can generate events that represent transaction boundaries and that enrich change event messages.
Limits on when Debezium receives transaction metadata
Debezium registers and receives metadata only for transactions that occur after you deploy the connector. Metadata for transactions that occur before you deploy the connector is not available. |
Debezium generates transaction boundary events for the BEGIN
and END
delimiters in every transaction.
Transaction boundary events contain the following fields:
status
-
BEGIN
orEND
. id
-
String representation of the unique transaction identifier composed of Informix transaction ID itself and LSN of given operation separated by colon, i.e. the format is
txID:LSN
. ts_ms
-
The time of a transaction boundary event (
BEGIN
orEND
event) at the data source. If the data source does not provide Debezium with the event time, then the field instead represents the time at which Debezium processes the event. event_count
(forEND
events)-
Total number of events emmitted by the transaction.
data_collections
(forEND
events)-
An array of pairs of
data_collection
andevent_count
elements that indicates the number of events that the connector emits for changes that originate from a data collection.
{
"status": "BEGIN",
"id": "571:53195829",
"ts_ms": 1486500577125,
"event_count": null,
"data_collections": null
}
{
"status": "END",
"id": "571:53195832",
"ts_ms": 1486500577691,
"event_count": 2,
"data_collections": [
{
"data_collection": "testdb.informix.tablea",
"event_count": 1
},
{
"data_collection": "testdb.informix.tableb",
"event_count": 1
}
]
}
By default, the connector emits transaction events to the <topic.prefix>
.transaction
topic.
You can override the default setting by changing the value of the topic.transaction
property.
When transaction metadata is enabled, the connector enriches the change event Envelope
with a new transaction
field.
This field provides information about every event in the form of a composite of fields:
id
-
String representation of unique transaction identifier.
total_order
-
The absolute position of the event among all events generated by the transaction.
data_collection_order
-
The per-data collection position of the event among all events that were emitted by the transaction.
Following is an example of a message:
{
"before": null,
"after": {
"pk": "2",
"aa": "1"
},
"source": {
...
},
"op": "c",
"ts_ms": "1580390884335",
"ts_us": "1580390884335641",
"ts_ns": "1580390884335641387",
"transaction": {
"id": "571:53195832",
"total_order": "1",
"data_collection_order": "1"
}
}
Data change events
The Debezium Informix connector generates a data change event for each row-level INSERT
, UPDATE
, and DELETE
operation.
Each event contains a key and a value.
The structure of the key and the value depends on the table that was changed.
Debezium and Kafka Connect are designed for processing continuous streams of event messages. However, because the structure of these events might change over time, consumers might encounter difficulties when processing some Debezium events. To address this challenge, each event is designed to be self-contained. That is, the event contains either the schema for its content, or, in environments that use a schema registry, a schema ID that the consumer can use to obtain the schema from the registry.
The JSON structure in the following example shows how a typical Debezium event record represents the four basic four components of change event.
The exact representation an event depends on the Kafka Connect converter that you configure for use with your application.
A schema
field is present in a change event only when you configure the converter to produce it.
Likewise, the event key and event payload are present a change event only if you configure the converter to produce them.
If you use the JSON converter, and you configure it to produce all four basic change event parts, change events have the following structure:
{
"schema": { (1)
...
},
"payload": { (2)
...
},
"schema": { (3)
...
},
"payload": { (4)
...
},
}
Item | Field name | Description |
---|---|---|
1 |
|
The first |
2 |
|
The first |
3 |
|
The second |
4 |
|
The second |
By default, the connector streams change event records to topics with names that are the same as the event’s originating table. For more information, see topic names.
The Debezium Informix connector ensures that all Kafka Connect schema names adhere to the Avro schema name format.
Conforming to the Avro schema name format means that the logical server name starts with a Latin letter, or with an underscore, that is, The use of underscores to replace invalid characters can lead to unexpected conflicts. For example, a conflict can result when the name of a logical server, a database, or a table contains one or more invalid characters, and those characters are the only characters that distinguish the name from the name of another entity of the same type. Naming conflicts can also occur, because the names of databases, schemas, and tables in Informix can be case-sensitive. In some cases, the connector might emit event records from more than one table to the same Kafka topic. |
Change event keys
A change event’s key contains the schema for the changed table’s key and the changed row’s actual key.
Both the schema and its corresponding payload contain a field for each column in the changed table’s PRIMARY KEY
(or unique constraint) at the time the connector created the event.
Consider the following customers
table, which is followed by an example of a change event key for this table.
CREATE TABLE customers (
id INTEGER IDENTITY(1001,1) NOT NULL PRIMARY KEY,
first_name VARCHAR(255) NOT NULL,
last_name VARCHAR(255) NOT NULL,
email VARCHAR(255) NOT NULL UNIQUE
);
When Debezium captures a change from the customers
table, it emits a change event record that contains the event key schema.
As long as the definition of the customers
table remains unchanged, every change that Debezium captures from the customers
table results in an event record that has the same key structure.
The following example shows a JSON representation of the event structure:
{
"schema": { (1)
"type": "struct",
"fields": [ (2)
{
"type": "int32",
"optional": false,
"field": "ID"
}
],
"optional": false, (3)
"name": "mydatabase.myschema.customers.Key" (4)
},
"payload": { (5)
"ID": 1004
}
}
Item | Field name | Description |
---|---|---|
1 |
|
The schema element of the key shows the Kafka Connect schema that describes the structure of the key’s |
2 |
|
Specifies each field that is expected in the |
3 |
|
Indicates whether the event key must contain a value in its |
4 |
|
Name of the schema that defines the structure of the key’s payload.
This schema describes the structure of the primary key for the table that was changed.
Key schema names have the following format:
In the preceding example the schema name is comprised of the following elements:
|
5 |
|
Contains the key of the table row in which the change event occurred.
In the preceding example, the key contains a single |
Change event values
The value in a change event is a bit more complicated than the key.
Like the event key, the value includes a schema
element and a payload
element.
The schema
element contains the schema that describes the Envelope
structure of the payload
element, including its nested fields.
Change events for operations that create, update or delete data all have a value payload with an envelope structure.
Consider the sample `customers`table that was used in the earlier example of a change event key:
CREATE TABLE customers (
id INTEGER IDENTITY(1001,1) NOT NULL PRIMARY KEY,
first_name VARCHAR(255) NOT NULL,
last_name VARCHAR(255) NOT NULL,
email VARCHAR(255) NOT NULL UNIQUE
);
The value
element of every change event that Debezium captures from the customers
table uses the same schema.
The payload of each event value varies according to the event type:
create events
The following example shows the value portion of a change event that the connector generates for an operation that creates data in the customers
table:
{
"schema": { (1)
"type": "struct",
"fields": [
{
"type": "struct",
"fields": [
{
"type": "int32",
"optional": false,
"field": "id"
},
{
"type": "string",
"optional": false,
"field": "first_name"
},
{
"type": "string",
"optional": false,
"field": "last_name"
},
{
"type": "string",
"optional": false,
"field": "email"
}
],
"optional": true,
"name": "mydatabase.myschema.customers.Value", (2)
"field": "before"
},
{
"type": "struct",
"fields": [
{
"type": "int32",
"optional": false,
"field": "id"
},
{
"type": "string",
"optional": false,
"field": "first_name"
},
{
"type": "string",
"optional": false,
"field": "last_name"
},
{
"type": "string",
"optional": false,
"field": "email"
}
],
"optional": true,
"name": "mydatabase.myschema.customers.Value",
"field": "after"
},
{
"type": "struct",
"fields": [
{
"type": "string",
"optional": false,
"field": "version"
},
{
"type": "string",
"optional": false,
"field": "connector"
},
{
"type": "string",
"optional": false,
"field": "name"
},
{
"type": "int64",
"optional": false,
"field": "ts_ms"
},
{
"type": "int64",
"optional": false,
"field": "ts_us"
},
{
"type": "int64",
"optional": false,
"field": "ts_ns"
},
{
"type": "boolean",
"optional": true,
"default": false,
"field": "snapshot"
},
{
"type": "string",
"optional": false,
"field": "db"
},
{
"type": "string",
"optional": false,
"field": "schema"
},
{
"type": "string",
"optional": false,
"field": "table"
},
{
"type": "string",
"optional": true,
"field": "commit_lsn"
},
{
"type": "string",
"optional": true,
"field": "change_lsn"
},
{
"type": "string",
"optional": true,
"field": "txId"
},
{
"type": "string",
"optional": true,
"field": "begin_lsn"
}
],
"optional": false,
"name": "io.debezium.connector.informix.Source", (3)
"field": "source"
},
{
"type": "string",
"optional": false,
"field": "op"
},
{
"type": "int64",
"optional": true,
"field": "ts_ms"
},
{
"type": "int64",
"optional": true,
"field": "ts_us"
},
{
"type": "int64",
"optional": true,
"field": "ts_ns"
}
],
"optional": false,
"name": "mydatabase.myschema.customers.Envelope" (4)
},
"payload": { (5)
"before": null, (6)
"after": { (7)
"id": 1005,
"first_name": "john",
"last_name": "doe",
"email": "john.doe@example.org"
},
"source": { (8)
"version": "2.6.2.Final",
"connector": "informix",
"name": "myconnector",
"ts_ms": 1559729468470,
"ts_us": 1559729468470000,
"ts_ns": 1559729468470000000,
"snapshot": false,
"db": "mydatabase",
"schema": "myschema",
"table": "customers",
"commit_lsn": "627404540760620",
"change_lsn": "627404540485812",
"txId": "157",
"begin_lsn": "627404540372400"
},
"op": "c", (9)
"ts_ms": 1559729471739, (10)
"ts_us": 1559729471739241, (10)
"ts_ns": 1559729471739241367 (10)
}
}
Item | Field name | Description | ||||||||
---|---|---|---|---|---|---|---|---|---|---|
1 |
|
The value’s schema, which describes the structure of the value’s payload. The schema of the value in a change event is the same in every change event that the connector generates for a particular table. |
||||||||
2 |
|
In the |
||||||||
3 |
|
|
||||||||
4 |
|
|
||||||||
5 |
|
The value’s actual data.
The payload provides the information about how an event changed data in a table row. |
||||||||
6 |
|
An optional field that represent the state of the row before an event occurs.
When the value of the |
||||||||
7 |
|
An optional field that specifies the state of the row after the event occurred.
In this example, the |
||||||||
8 |
|
Mandatory field that describes the source metadata for the event.
The
|
||||||||
9 |
|
Mandatory string that describes the type of operation that caused the connector to generate the event.
In the preceding example,
|
||||||||
10 |
|
Optional field that displays the time at which the connector processed the event.
The time is based on the system clock in the JVM that runs the Kafka Connect task. |
update events
The value of a change event for an update in the sample customers
table has the same schema as a create event for that table.
Similarly, the payload of the value of an update event has a structure that is mirrors the structure of the value payload in a create event.
However, the value
payloads of_update_ events and create events do not include the same values.
The following example shows the change event value for an event record that the connector generates in response to an update in the customers
table:
{
"schema": { ... },
"payload": {
"before": { (1)
"id": 1005,
"first_name": "john",
"last_name": "doe",
"email": "john.doe@example.org"
},
"after": { (2)
"ID": 1005,
"first_name": "john",
"last_name": "doe",
"email": "noreply@example.org"
},
"source": { (3)
"version": "2.6.2.Final",
"connector": "informix",
"name": "myconnector",
"ts_ms": 1559729995937,
"ts_us": 1559729995937000,
"ts_ns": 1559729995937000000,
"snapshot": false,
"db": "mydatabase",
"schema": "myschema",
"table": "customers",
"commit_lsn": "627404540760620",
"change_lsn": "627404540485812",
"txId": "157",
"begin_lsn": "627404540372400"
},
"op": "u", (4)
"ts_ms": 1559729998706, (5)
"ts_us": 1559729998706742, (5)
"ts_ns": 1559729998706742877 (5)
}
}
Item | Field name | Description |
---|---|---|
1 |
|
An optional field that specifies the state of the row before an event occurs.
In an update event value, the |
2 |
|
An optional field that specifies the state of a row after an event occurs.
By comparing the |
3 |
|
Mandatory field that describes the source metadata for the event.
The
|
4 |
|
Mandatory string that describes the type of operation.
In an update event value, the |
5 |
|
Optional field that displays the time at which the connector processed the event.
The time is based on the system clock in the JVM that runs the Kafka Connect task. |
If you update the columns for a row’s primary or unique key, you change the value of the row’s key. After a key change, Debezium emits the following events:
|
delete events
The value
in a delete change event for a table has a schema
portion that is similar to the schema
element in create and update events for the same table.
After a user performs a delete operation in the sample customers
table, Debezium emits an event message such as the one in the following example:
{
"schema": { ... },
},
"payload": {
"before": { (1)
"id": 1005,
"first_name": "john",
"last_name": "doe",
"email": "noreply@example.org"
},
"after": null, (2)
"source": { (3)
"version": "2.6.2.Final",
"connector": "informix",
"name": "myconnector",
"ts_ms": 1559730445243,
"ts_us": 1559730445243000,
"ts_ns": 1559730445243000000,
"snapshot": false,
"db": "mydatabase",
"schema": "myschema",
"table": "customers",
"commit_lsn": "627404540760620",
"change_lsn": "627404540485812",
"txId": "157",
"begin_lsn": "627404540372400"
},
"op": "d", (4)
"ts_ms": 1559730450205, (5)
"ts_us": 1559730450205104, (5)
"ts_ns": 1559730450205104870 (5)
}
}
Item | Field name | Description |
---|---|---|
1 |
|
Optional field that specifies the state of the row before the event occurred.
In a delete event value, the |
2 |
|
Optional field that specifies the state of the row after the event occurred.
In a delete event value, the |
3 |
|
Mandatory field that describes the source metadata for the event.
In a delete event value, the
|
4 |
|
Mandatory string that describes the type of operation.
The value of the |
5 |
|
Optional field that displays the time at which the connector processed the event.
The time is based on the system clock in the JVM running the Kafka Connect task. |
A delete change event record provides a consumer with the information that it needs to process the removal of the row. The record includes the previous values to support consumers that might require them to process the removal.
Informix connector events are designed to work with Kafka log compaction. Log compaction enables removal of some older messages as long as at least the most recent message for every key is kept. Retaining the most recent message enables Kafka to reclaim storage space while ensuring that the topic contains a complete data set that can be used for reloading key-based state.
When a row is deleted, the delete event value still works with log compaction, because Kafka can remove all earlier messages that have that same key.
However, for Kafka to remove all messages that have that same key, the message value must be null
.
To make this possible, after Debezium’s Informix connector emits a delete event, the connector emits a special tombstone event that has the same key but a null
value.
Data type mappings
For a complete description of the data types that Informix supports, see Data Types in the Informix documentation.
The Informix connector represents changes to rows by emitting events whose structures mirror the structure of the source tables in which the change events occur. Event records contain fields for each column value. To populate values in these fields from the source columns, the connector uses a default mapping to convert the values from the original Informix data types to a Kafka Connect schema type or a semantic type. The connector provides default mappings for the following Informix data types:
-
Decimal types If the default data type conversions do not meet your needs, you can create a custom converter for the connector.
Basic types
The following table describes how the connector maps each Informix data type to a literal type and a semantic type in event fields.
-
literal type describes how the value is represented using Kafka Connect schema types:
INT8
,INT16
,INT32
,INT64
,FLOAT32
,FLOAT64
,BOOLEAN
,STRING
,BYTES
,ARRAY
,MAP
, andSTRUCT
. -
semantic type describes how the Kafka Connect schema captures the meaning of the field using the name of the Kafka Connect schema for the field.
Informix data type | Literal type (schema type) | Semantic type (schema name) and Notes |
---|---|---|
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
|
|
|
|
|
|
|
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
|
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
n/a |
|
|
8-bit unsigned integer value between 0 and 255, thus needs to be stored as int16 |
|
|
n/a |
|
|
n/a |
If present, a column’s default value is propagated to the corresponding field’s Kafka Connect schema. Change events contain the field’s default value unless an explicit column value is specified. Consequently, there is rarely a need to obtain the default value from the schema. Passing the default value helps satisfy compatibility rules when using Avro as the serialization format together with the Confluent schema registry.
Temporal types
Informix maps temporal types based on the value of the time.precision.mode
connector configuration property.
The following sections describe these mappings:
time.precision.mode=adaptive
To ensure that events exactly represent the values in the database, when the time.precision.mode
configuration property is set to the default value, adaptive
, the connector determines the literal and semantic types based on the column’s data type definition.
Informix data type | Literal type (schema type) | Semantic type (schema name) and Notes |
---|---|---|
|
|
|
|
|
|
time.precision.mode=connect
When the time.precision.mode
configuration property is set to connect
, the connector uses Kafka Connect logical types.
This setting can be useful for consumers that can handle only the built-in Kafka Connect logical types, and that cannot handle variable-precision time values.
However, because Informix supports tens of microsecond precision, if a connector is configured to use connect
time precision, and the database column has a fractional second precision value that is greater than 3, the connector generates events that result in a loss of precision.
Informix data type | Literal type (schema type) | Semantic type (schema name) and Notes |
---|---|---|
|
|
|
|
|
|
Timestamp types
The DATETIME
type represents a timestamp without time zone information.
Such columns are converted into an equivalent Kafka Connect value based on UTC.
For example, the DATETIME
value "2018-06-20 15:13:16.94514" is represented by an io.debezium.time.Timestamp
with the value "1529507596000".
The timezone of the JVM running Kafka Connect and Debezium does not affect this conversion.
Decimal types
Informix data type | Literal type (schema type) | Semantic type (schema name) and Notes |
---|---|---|
|
|
|
|
|
+
The |
Setting up Informix
For Debezium to capture change events that are committed to Informix tables, a Informix database administrator with the necessary privileges must configure the database for change data capture.
Perform the following tasks to prepare for using the Change Data Capture API:
-
As the database user
informix
, run thesyscdcv1.sql
script from the$INFORMIXDIR/etc
directory. This will install thesyscdcv1
database. -
Verify that the
syscdcv1
database exists by creating a connection to it as userinformix
. -
Set the
DB_LOCALE
environment variable to be the same as the locale of the database from which you want to capture data.
Specific guidance about optimizing Informix for change data capture is beyond the scope of this documentation. |
Deployment
To deploy a Debezium Informix connector, you install the Debezium Informix connector archive, configure the connector, and start the connector by adding its configuration to Kafka Connect.
-
Apache ZooKeeper, Apache Kafka, and Kafka Connect are installed.
-
Informix is installed and capture mode is enabled for tables to prepare the database to be used with the Debezium connector.
-
Download the Debezium Informix connector plug-in archive from Maven Central.
-
Extract the JAR files into your Kafka Connect environment.
-
Download the JDBC driver for Informix and Informix Change Stream client from Maven Central, and copy the downloaded JAR files to the directory that contains the Debezium Informix connector JAR file (that is,
debezium-connector-informix-2.6.2.Final.jar
).Due to licensing requirements, the Debezium Informix connector archive does not include the Informix JDBC driver and Change Stream client that Debezium requires to connect to a Informix database. To enable the connector to access the database, you must add the driver and client library to your connector environment.
-
Add the directory with the JAR files to Kafka Connect’s
plugin.path
. -
Restart your Kafka Connect process to pick up the new JAR files.
If you are working with immutable containers, see Debezium’s container images for Apache ZooKeeper, Apache Kafka and Kafka Connect with the Informix connector already installed and ready to run.
You can also run Debezium on Kubernetes and OpenShift.
Informix connector configuration example
The following example shows the configuration for a connector instance that captures data from an Informix server with the logical name fullfillment
on port 9088 at 192.168.99.100.
Typically, you configure the Debezium Informix connector in a JSON file by setting the configuration properties that are available for the connector.
You can choose to produce events for a subset of the schemas and tables in a database. Optionally, you can ignore, mask, or truncate columns that contain sensitive data, that are larger than a specified size, or that you do not need.
{
"name": "informix-connector", (1)
"config": {
"connector.class": "io.debezium.connector.informix.InformixConnector", (2)
"database.hostname": "192.168.99.100", (3)
"database.port": "9088", (4)
"database.user": "informix", (5)
"database.password": "in4mix", (6)
"database.dbname": "mydatabase", (7)
"topic.prefix": "fullfillment", (8)
"table.include.list": "mydatabase.myschema.customers", (9)
"schema.history.internal.kafka.bootstrap.servers": "kafka:9092", (10)
"schema.history.internal.kafka.topic": "schemahistory.fullfillment" (11)
}
}
1 | The name of the connector when registered with a Kafka Connect service. |
2 | The name of this Informix connector class. |
3 | The address of the Informix instance. |
4 | The port number of the Informix instance. |
5 | The name of the Informix user. |
6 | The password for the Informix user. |
7 | The name of the database to capture changes from. |
8 | The logical name of the Informix instance/cluster, which forms a namespace and is used in all the names of the Kafka topics to which the connector writes, the Kafka Connect schema names, and the namespaces of the corresponding Avro schema when the Avro Connector is used. |
9 | A list of all tables whose changes Debezium should capture. |
10 | The list of Kafka brokers that this connector uses to write and recover DDL statements to the database schema history topic. |
11 | The name of the database schema history topic where the connector writes and recovers DDL statements. This topic is for internal use only and should not be used by consumers. |
For the complete list of the configuration properties that you can set for the Debezium Informix connector, see Informix connector properties.
You can send this configuration with a POST
command to a running Kafka Connect service.
The service records the configuration and starts one connector task that performs the following actions:
-
Connects to the Informix database.
-
Reads change-data tables for tables that are in capture mode.
-
Streams change event records to Kafka topics.
Adding connector configuration
To start running a Informix connector, create a connector configuration and add the configuration to your Kafka Connect cluster.
-
Informix replication is enabled to expose change data for tables that are in capture mode.
-
The Informix connector is installed.
-
Create a configuration for the Informix connector.
-
Use the Kafka Connect REST API to add that connector configuration to your Kafka Connect cluster.
After the connector starts, it performs a consistent snapshot of the Informix database tables that the connector is configured to capture changes for. The connector then starts generating data change events for row-level operations and streaming change event records to Kafka topics.
Connector properties
The Debezium Informix connector has numerous configuration properties that you can use to achieve the right connector behavior for your application. Many properties have default values. Information about the properties is organized as follows:
-
Database schema history connector configuration properties that control how Debezium processes events that it reads from the database schema history topic.
-
Pass-through database driver properties that control the behavior of the database driver.
Required Debezium Informix connector configuration properties
The following configuration properties are required unless a default value is available.
Property | Default | Description | ||
---|---|---|---|---|
No default |
Unique name for the connector. You can register a connector with the specified name only once. Subsequent attempts result in failures. This property is required by all Kafka Connect connectors. |
|||
No default |
The name of the Java class for the connector.
Always use the value |
|||
|
The maximum number of tasks that this connector can create. The Informix connector always uses a single task and therefore does not use this value, so the default is always acceptable. |
|||
No default |
IP address or hostname of the Informix database server. |
|||
|
Integer port number of the Informix database server. |
|||
No default |
Name of the Informix database user for connecting to the Informix database server. |
|||
No default |
Password to use when connecting to the Informix database server. |
|||
No default |
The name of the Informix database from which to stream changes. |
|||
No default |
Topic prefix which provides a namespace for the particular Informix database server that hosts the database for which Debezium is capturing changes.
Only alphanumeric characters, hyphens, dots and underscores must be used in the topic prefix name.
The topic prefix is used for all Kafka topics that receive records from this connector.
Specify a value that is unique across all connectors in the Kafka Connect deployment.
|
|||
No default |
An optional, comma-separated list of regular expressions that match fully-qualified table identifiers for tables whose changes you want the connector to capture.
When this property is set, the connector captures changes only from the specified tables.
Each identifier is of the form databaseName.schemaName.tableName.
By default, the connector captures changes in every non-system table. To match the name of a table, Debezium applies the regular expression that you specify as an anchored regular expression.
That is, the specified expression is matched against the entire name string of the table it does not match substrings that might be present in a table name. |
|||
No default |
An optional, comma-separated list of regular expressions that match fully-qualified table identifiers for tables whose changes you do not want the connector to capture.
The connector captures changes in each non-system table that is not included in the exclude list.
Each identifier is of the form databaseName.schemaName.tableName. To match the name of a table, Debezium applies the regular expression that you specify as an anchored regular expression.
That is, the specified expression is matched against the entire name string of the table it does not match substrings that might be present in a table name. |
|||
empty string |
An optional, comma-separated list of regular expressions that match the fully-qualified names of columns to include in change event record values.
The fully-qualified name of a column observes one of the following formats: databaseName.tableName.columnName, or databaseName.schemaName.tableName.columnName. To match the name of a column, Debezium applies the regular expression that you specify as an anchored regular expression.
That is, the specified expression is matched against the entire name string of the column; it does not match substrings that might be present in a column name.
If you include this property in the configuration, do not also set the |
|||
empty string |
An optional, comma-separated list of regular expressions that match the fully-qualified names of columns to exclude from change event values.
The fully-qualified name of a column observes one of the following formats: databaseName.tableName.columnName, or databaseName.schemaName.tableName.columnName. To match the name of a column, Debezium applies the regular expression that you specify as an anchored regular expression.
That is, the specified expression is matched against the entire name string of the column; it does not match substrings that might be present in a column name.
Primary key columns are always included in the event’s key, the value that you set in this property would exclude them.
If you include this property in the configuration, do not set the |
|||
n/a |
An optional, comma-separated list of regular expressions that match the fully-qualified names of character-based columns.
The fully-qualified name of a column observes one of the following formats: databaseName.tableName.columnName, or databaseName.schemaName.tableName.columnName. A pseudonym consists of the hashed value that results from applying the specified hashAlgorithm and salt.
Based on the hash function that is used, referential integrity is maintained, while column values are replaced with pseudonyms.
Supported hash functions are described in the MessageDigest section of the Java Cryptography Architecture Standard Algorithm Name Documentation. column.mask.hash.SHA-256.with.salt.CzQMA0cB5K = inventory.orders.customerName, inventory.shipment.customerName If necessary, the pseudonym is automatically shortened to the length of the column.
The connector configuration can include multiple properties that specify different hash algorithms and salts. |
|||
|
Specifies the numeric precision that the connector uses to represent time, date, and timestamps values. |
|||
|
Specifies whether a delete event is followed by a tombstone event. |
|||
|
Boolean value that specifies whether the connector publishes changes in the database schema to the Kafka topic that has the same name as the database server ID.
When the default value is set, schema changes are recorded with a |
|||
n/a |
An optional, comma-separated list of regular expressions that match the fully-qualified names of character-based columns.
Set this property if you want to truncate the data in a set of columns when it exceeds the number of characters specified by the length in the property name.
Set The fully-qualified name of a column observes one of the following formats: databaseName.tableName.columnName, or databaseName.schemaName.tableName.columnName. You can specify multiple properties with different lengths in a single configuration. |
|||
n/a |
An optional, comma-separated list of regular expressions that match the fully-qualified names of character-based columns.
Set this property if you want the connector to mask the values for a set of columns, for example, if they contain sensitive data.
Set The fully-qualified name of a column observes one of the following formats: databaseName.tableName.columnName, or databaseName.schemaName.tableName.columnName. You can specify multiple properties with different lengths in a single configuration. |
|||
n/a |
An optional, comma-separated list of regular expressions that match the fully-qualified names of columns for which you want the connector to emit extra parameters that represent column metadata. When this property is set, the connector adds the following fields to the schema of event records:
These parameters propagate a column’s original type name and length (for variable-width types), respectively. The fully-qualified name of a column observes one of the following formats: databaseName.tableName.columnName, or databaseName.schemaName.tableName.columnName. |
|||
n/a |
An optional, comma-separated list of regular expressions that specify the fully-qualified names of data types that are defined for columns in a database. When this property is set, for columns with matching data types, the connector emits event records that include the following extra fields in their schema:
These parameters propagate the name of a column’s original data type, and, for variable-width types, its length, and scale. The fully-qualified name of a column observes one of the following formats: databaseName.tableName.typeName, or databaseName.schemaName.tableName.typeName. For the list of Informix-specific data type names, see the Informix data type mappings . |
|||
empty string |
A list of expressions that specify the columns that the connector uses to form custom message keys for change event records that it publishes to the Kafka topics for specified tables. By default, Debezium uses the primary key column of a table as the message key for records that it emits.
In place of the default, or to specify a key for tables that lack a primary key, you can configure custom message keys based on one or more columns.
The property can list entries for multiple tables.
Use a semicolon to separate entries for different tables in the list. |
|||
none |
Specifies how to adjust schema names for compatibility with the message converter that the connector uses.
|
|||
none |
SSpecifies how to adjust field names for compatibility with the message converter that the connector uses.
For more information about Avro compatibility, see Avro naming . |
Advanced connector configuration properties
The following advanced configuration properties have defaults that work in most situations and therefore rarely need to be specified in the connector’s configuration.
Property | Default | Description | ||
---|---|---|---|---|
No default |
Enumerates a comma-separated list of the symbolic names of the custom converter instances that the connector can use.
For example,
You must set the For each converter that you configure for a connector, you must also add a
For example, isbn.type: io.debezium.test.IsbnConverter If you want to further control the behavior of a configured converter, you can add one or more configuration parameters to pass values to the converter.
To associate any additional configuration parameter with a converter, prefix the parameter names with the symbolic name of the converter. isbn.schema.name: io.debezium.informix.type.Isbn |
|||
initial |
Specifies the criteria for performing a snapshot when the connector starts:
For more information, see custom snapshotter SPI. |
|||
false |
If the |
|||
false |
If the |
|||
false |
If the |
|||
false |
If the |
|||
false |
If the |
|||
No default |
If |
|||
exclusive |
Controls whether and for how long the connector holds a table lock. Table locks prevent other database clients from performing certain table operations during a snapshot. You can set the following values:
|
|||
No default |
When |
|||
|
Specifies how the connector queries data while performing a snapshot.
This setting enables you to manage snapshot content in a more flexible manner compared to using the |
|||
No default |
When |
|||
|
During a snapshot, specifies the transaction isolation level and the length of time that the connector locks tables that are in capture mode. Specify one of the following values:
|
|||
|
Positive integer value that specifies the timeout behavior of a read call to the change stream client. +
|
|||
|
Positive integer value that specifies the maximum size of each batch of records that the Informix Change Stream Client processes. |
|||
|
Specifies how the connector handles exceptions during processing of events.
|
|||
|
Positive integer value that specifies the number of milliseconds that the connector waits for new change events to appear before it starts processing a batch of events. |
|||
|
Positive integer value that specifies the maximum size of each batch of events that the connector processes. |
|||
|
Positive integer value that specifies the maximum number of records that the blocking queue can hold.
When Debezium reads events streamed from the database, it places the events in the blocking queue before it writes them to Kafka.
The blocking queue can provide backpressure for reading change events from the database
in cases where the connector ingests messages faster than it can write them to Kafka, or when Kafka becomes unavailable.
Events that are held in the queue are disregarded when the connector periodically records offsets.
Always set the value of |
|||
|
A long integer value that specifies the maximum volume of the blocking queue in bytes.
By default, volume limits are not specified for the blocking queue.
To specify the number of bytes that the queue can consume, set this property to a positive long value. |
|||
|
Controls how frequently the connector sends heartbeat messages to a Kafka topic. The default behavior is that the connector does not send heartbeat messages. |
|||
No default |
An interval in milliseconds that the connector should wait before performing a snapshot when the connector starts. If you start multiple connectors in a cluster, this property is useful for avoiding snapshot interruptions, which might cause re-balancing of connectors. |
|||
All tables specified in |
An optional, comma-separated list of regular expressions that match the fully-qualified names ( To match the name of a table, Debezium applies the regular expression that you specify as an anchored regular expression. That is, the specified expression is matched against the entire name string of the table; it does not match substrings that might be present in a table name. |
|||
|
During a snapshot, the connector reads table content in batches of rows. This property specifies the maximum number of rows in a batch. |
|||
|
Specifies the maximum amount of time (in milliseconds) to wait to obtain table locks when performing a snapshot.
If the connector cannot acquire table locks during this interval, the snapshot fails.
For more information, see How the connector performs snapshots.
|
|||
No default |
Specifies the table rows to include in a snapshot. Use the property if you want a snapshot to include only a subset of the rows in a table. This property affects snapshots only. It does not apply to events that the connector reads from the log during the streaming phase. The property contains a comma-separated list of fully-qualified table names in the form From a "snapshot.select.statement.overrides": "mydatabase.customer.orders", "snapshot.select.statement.overrides.mydatabase.customer.orders": "SELECT * FROM [customers].[orders] WHERE delete_flag = 0 ORDER BY id DESC" In the resulting snapshot, the connector includes only the records for which |
|||
|
Determines whether the connector generates events with transaction boundaries and enriches change event envelopes with transaction metadata.
Set the value to |
|||
|
A comma-separated list of operation types that the connector skips during streaming. You can specify the following values:
|
|||
No default |
Fully-qualified name of the data collection that is used to send signals to the connector.
Use the following format to specify the collection name: |
|||
|
List of the signaling channel names that are enabled for the connector. By default, the following channels are available:
|
|||
No default |
List of the notification channel names that are enabled for the connector. By default, the following channels are available:
|
|||
|
The maximum number of rows that the connector fetches and reads into memory during an incremental snapshot chunk. Increasing the chunk size provides greater efficiency, because the snapshot runs fewer snapshot queries of a greater size. However, larger chunk sizes also require more memory to buffer the snapshot data. Adjust the chunk size to a value that provides the best performance in your environment. |
|||
|
Specifies the watermarking mechanism that the connector uses during an incremental snapshot to deduplicate events that might be captured by an incremental snapshot and then recaptured after streaming resumes.
|
|||
|
The name of the |
|||
|
Specifies the delimiter that the connector uses to construct topic names. |
|||
|
The cache size allocated for storing topic names in a bounded concurrent hash map. This cache helps to determine the topic name that corresponds to a given data collection. |
|||
|
Specifies a string that the connector appends to the name of the topic to which it sends heartbeat messages.
The topic name has the following pattern: |
|||
|
Specifies a string that the connector appends to the name of the topic to which it sends transaction metadata messages.
The topic name has the following pattern: |
|||
|
Specifies the maximum number of threads that the connector uses when performing an initial snapshot. To enable parallel initial snapshots, set the property to a value greater than 1. In a parallel initial snapshot, the connector processes multiple tables concurrently. This feature is incubating. |
|||
|
Defines tags for customizing MBean object names by adding metadata that provides contextual information that enables you to organize and categorize metrics data.
Specify a comma-separated list of key-value pairs.
Each key represents a tag for the MBean object name, and the corresponding value represents a value for the key, for example, The connector appends the specified tags to the base MBean object name. You can define tags to identify particular application instances, environments, regions, versions, and so forth. |
|||
|
The maximum number of retries on retriable errors (e.g. connection errors) before failing (-1 = no limit, 0 = disabled, > 0 = num of retries). |
Debezium connector database schema history configuration properties
Debezium provides a set of schema.history.internal.*
properties that control how the connector interacts with the schema history topic.
The following table describes the schema.history.internal
properties for configuring the Debezium connector.
Property | Default | Description | ||
---|---|---|---|---|
No default |
The full name of the Kafka topic where the connector stores the database schema history. |
|||
No default |
A list of host/port pairs that the connector uses for establishing an initial connection to the Kafka cluster. This connection is used for retrieving the database schema history previously stored by the connector, and for writing each DDL statement read from the source database. Each pair should point to the same Kafka cluster used by the Kafka Connect process. |
|||
|
An integer value that specifies the maximum number of milliseconds the connector should wait during startup/recovery while polling for persisted data. The default is 100ms. |
|||
|
An integer value that specifies the maximum number of milliseconds the connector should wait while fetching cluster information using Kafka admin client. |
|||
|
An integer value that specifies the maximum number of milliseconds the connector should wait while create kafka history topic using Kafka admin client. |
|||
|
The maximum number of times that the connector should try to read persisted history data before the connector recovery fails with an error. The maximum amount of time to wait after receiving no data is |
|||
|
A Boolean value that specifies whether the connector should ignore malformed or unknown database statements or stop processing so a human can fix the issue.
The safe default is |
|||
|
A Boolean value that specifies whether the connector records schema structures from all tables in a schema or database, or only from tables that are designated for capture.
|
|||
|
A Boolean value that specifies whether the connector records schema structures from all logical databases in the database instance.
|
Debezium relies on a Kafka producer to write schema changes to database schema history topics.
Similarly, it relies on a Kafka consumer to read from database schema history topics when a connector starts.
You define the configuration for the Kafka producer and consumer clients by assigning values to a set of pass-through configuration properties that begin with the schema.history.internal.producer.*
and schema.history.internal.consumer.*
prefixes.
The pass-through producer and consumer database schema history properties control a range of behaviors, such as how these clients secure connections with the Kafka broker, as shown in the following example:
schema.history.internal.producer.security.protocol=SSL
schema.history.internal.producer.ssl.keystore.location=/var/private/ssl/kafka.server.keystore.jks
schema.history.internal.producer.ssl.keystore.password=test1234
schema.history.internal.producer.ssl.truststore.location=/var/private/ssl/kafka.server.truststore.jks
schema.history.internal.producer.ssl.truststore.password=test1234
schema.history.internal.producer.ssl.key.password=test1234
schema.history.internal.consumer.security.protocol=SSL
schema.history.internal.consumer.ssl.keystore.location=/var/private/ssl/kafka.server.keystore.jks
schema.history.internal.consumer.ssl.keystore.password=test1234
schema.history.internal.consumer.ssl.truststore.location=/var/private/ssl/kafka.server.truststore.jks
schema.history.internal.consumer.ssl.truststore.password=test1234
schema.history.internal.consumer.ssl.key.password=test1234
Debezium strips the prefix from the property name before it passes the property to the Kafka client.
See the Kafka documentation for more details about Kafka producer configuration properties and Kafka consumer configuration properties.
Debezium connector Kafka signals configuration properties
Debezium provides a set of signal.*
properties that control how the connector interacts with the Kafka signals topic.
The following table describes the Kafka signal
properties.
Property | Default | Description | ||
---|---|---|---|---|
<topic.prefix>-signal |
The name of the Kafka topic that the connector monitors for ad hoc signals.
|
|||
kafka-signal |
The name of the group ID that is used by Kafka consumers. |
|||
No default |
A list of host/port pairs that the connector uses for establishing an initial connection to the Kafka cluster. Each pair references the Kafka cluster that is used by the Debezium Kafka Connect process. |
|||
|
An integer value that specifies the maximum number of milliseconds that the connector waits when polling signals. |
|||
|
Enable the offset commit for the signal topic in order to guarantee At-Least-Once delivery. If disabled, only signals received when the consumer is up&running are processed. Any signals received when the consumer is down are lost. |
Debezium connector pass-through signals Kafka consumer client configuration properties
The Debezium connector provides for pass-through configuration of the signals Kafka consumer.
Pass-through signals properties begin with the prefix signals.consumer.*
.
For example, the connector passes properties such as signal.consumer.security.protocol=SSL
to the Kafka consumer.
Debezium strips the prefixes from the properties before it passes the properties to the Kafka signals consumer.
Debezium connector sink notifications configuration properties
The following table describes the notification
properties.
Property | Default | Description |
---|---|---|
No default |
The name of the topic that receives notifications from Debezium.
This property is required when you configure the |
Debezium connector pass-through database driver configuration properties
The Debezium connector provides for pass-through configuration of the database driver.
Pass-through database properties begin with the prefix driver.*
.
For example, the connector passes properties such as driver.foobar=false
to the JDBC URL.
As is the case with the pass-through properties for database schema history clients, Debezium strips the prefixes from the properties before it passes them to the database driver.
Monitoring
The Debezium Informix connector provides three types of metrics that are in addition to the built-in support for JMX metrics that Apache ZooKeeper, Apache Kafka, and Kafka Connect provide.
-
Snapshot metrics provide information about connector operation while performing a snapshot.
-
Streaming metrics provide information about connector operation when the connector is capturing changes and streaming change event records.
-
Schema history metrics provide information about the status of the connector’s schema history.
Debezium monitoring documentation provides details for how to expose these metrics by using JMX.
Snapshot metrics
The MBean is debezium.informix:type=connector-metrics,context=snapshot,server=<topic.prefix>
.
Snapshot metrics are not exposed unless a snapshot operation is active, or if a snapshot has occurred since the last connector start.
The following table lists the shapshot metrics that are available.
Attributes | Type | Description |
---|---|---|
|
The last snapshot event that the connector has read. |
|
|
The number of milliseconds since the connector has read and processed the most recent event. |
|
|
The total number of events that this connector has seen since last started or reset. |
|
|
The number of events that have been filtered by include/exclude list filtering rules configured on the connector. |
|
|
The list of tables that are captured by the connector. |
|
|
The length the queue used to pass events between the snapshotter and the main Kafka Connect loop. |
|
|
The free capacity of the queue used to pass events between the snapshotter and the main Kafka Connect loop. |
|
|
The total number of tables that are being included in the snapshot. |
|
|
The number of tables that the snapshot has yet to copy. |
|
|
Whether the snapshot was started. |
|
|
Whether the snapshot was paused. |
|
|
Whether the snapshot was aborted. |
|
|
Whether the snapshot completed. |
|
|
The total number of seconds that the snapshot has taken so far, even if not complete. Includes also time when snapshot was paused. |
|
|
The total number of seconds that the snapshot was paused. If the snapshot was paused several times, the paused time adds up. |
|
|
Map containing the number of rows scanned for each table in the snapshot. Tables are incrementally added to the Map during processing. Updates every 10,000 rows scanned and upon completing a table. |
|
|
The maximum buffer of the queue in bytes. This metric is available if |
|
|
The current volume, in bytes, of records in the queue. |
The connector also provides the following additional snapshot metrics when an incremental snapshot is executed:
Attributes | Type | Description |
---|---|---|
|
The identifier of the current snapshot chunk. |
|
|
The lower bound of the primary key set defining the current chunk. |
|
|
The upper bound of the primary key set defining the current chunk. |
|
|
The lower bound of the primary key set of the currently snapshotted table. |
|
|
The upper bound of the primary key set of the currently snapshotted table. |
Streaming metrics
The MBean is debezium.informix:type=connector-metrics,context=streaming,server=<topic.prefix>
.
The following table lists the streaming metrics that are available.
Attributes | Type | Description |
---|---|---|
|
The last streaming event that the connector has read. |
|
|
The number of milliseconds since the connector has read and processed the most recent event. |
|
|
The total number of events that this connector has seen since the last start or metrics reset. |
|
|
The total number of create events that this connector has seen since the last start or metrics reset. |
|
|
The total number of update events that this connector has seen since the last start or metrics reset. |
|
|
The total number of delete events that this connector has seen since the last start or metrics reset. |
|
|
The number of events that have been filtered by include/exclude list filtering rules configured on the connector. |
|
|
The list of tables that are captured by the connector. |
|
|
The length the queue used to pass events between the streamer and the main Kafka Connect loop. |
|
|
The free capacity of the queue used to pass events between the streamer and the main Kafka Connect loop. |
|
|
Flag that denotes whether the connector is currently connected to the database server. |
|
|
The number of milliseconds between the last change event’s timestamp and the connector processing it. The values will incoporate any differences between the clocks on the machines where the database server and the connector are running. |
|
|
The number of processed transactions that were committed. |
|
|
The coordinates of the last received event. |
|
|
Transaction identifier of the last processed transaction. |
|
|
The maximum buffer of the queue in bytes. This metric is available if |
|
|
The current volume, in bytes, of records in the queue. |
Schema evolution
While a Debezium Informix connector can capture schema changes, to update a schema, you must collaborate with a database administrator to ensure that the connector continues to produce change events.
When you initiate a schema update on a table, you must permit the update procedure to complete before you perform a new schema update on the same table. When possible, it is best to execute all DDLs in a single batch and perform the schema update procedure only once. |
Offline schema update
Informix does not support online schema updates while capturing changes. You must stop the Debezium Informix connector before you perform a schema update.
Because you must stop Debezium to complete the schema update procedure, to minimize disruptions to downstream applications, it’s best to perform this operation during a scheduled maintenance window. |
-
One or more tables that are in capture mode require schema updates.
-
Suspend the application that updates the database.
-
Wait for the Debezium connector to stream all unstreamed change event records.
-
Stop the Debezium connector.
-
Apply all changes to the source table schema.
-
Resume the application that updates the database.
-
Restart the Debezium connector.