| Modifier and Type | Class and Description |
|---|---|
static class | SourceTask.TransactionBoundaryRepresents the permitted values for the TRANSACTION_BOUNDARY_CONFIG property. |
| Modifier and Type | Field and Description |
|---|---|
static String | TRANSACTION_BOUNDARY_CONFIGThe configuration key that determines how source tasks will define transaction boundaries when exactly-once support is enabled. |
| Constructor and Description |
|---|
SourceTask() |
| Modifier and Type | Method and Description |
|---|---|
void | commit()This method is invoked periodically when offsets are committed for this source task. |
void | commitRecord(SourceRecord record)Deprecated. Use commitRecord(SourceRecord, RecordMetadata) instead. |
void | commitRecord(SourceRecord record, org.apache.kafka.clients.producer.RecordMetadata metadata) Commit an individual SourceRecord when the callback from the producer client is received. |
void | initialize(SourceTaskContext context)Initialize this SourceTask with the specified context object. |
abstract List<SourceRecord> | poll()Poll this source task for new records. |
abstract void | start(Map<String,String> props)Start the Task. |
abstract void | stop()Signal this SourceTask to stop. |
public static final String TRANSACTION_BOUNDARY_CONFIG
public void initialize(SourceTaskContext context)
public abstract void start(Map<String,String> props)
public abstract List<SourceRecord> poll() throws InterruptedException
null) in order for the task to transition to the PAUSED state if requested to do so. The task will be stopped on a separate thread, and when that happens this method is expected to unblock, quickly finish up any remaining processing, and return.
InterruptedExceptionpublic void commit()
throws InterruptedExceptionpoll(). Also see commitRecord(SourceRecord, RecordMetadata) which allows for a more fine-grained tracking of records that have been successfully delivered. SourceTasks are not required to implement this functionality; Kafka Connect will record offsets automatically. This hook is provided for systems that also need to store offsets internally in their own system.
InterruptedExceptionpublic abstract void stop()
poll() and commit(). For example, if a task uses a Selector to receive data over the network, this method could set a flag that will force poll() to exit immediately and invoke wakeup() to interrupt any ongoing requests.
@Deprecated public void commitRecord(SourceRecord record) throws InterruptedException
commitRecord(SourceRecord, RecordMetadata) instead. Commit an individual SourceRecord when the callback from the producer client is received. This method is also called when a record is filtered by a transformation, and thus will never be ACK'd by a broker.
This is an alias for commitRecord(SourceRecord, RecordMetadata) for backwards compatibility. The default implementation of commitRecord(SourceRecord, RecordMetadata) just calls this method. It is not necessary to override both methods.
SourceTasks are not required to implement this functionality; Kafka Connect will record offsets automatically. This hook is provided for systems that also need to store offsets internally in their own system.
record - SourceRecord that was successfully sent via the producer or filtered by a transformationInterruptedExceptionpublic void commitRecord(SourceRecord record, org.apache.kafka.clients.producer.RecordMetadata metadata) throws InterruptedException
Commit an individual SourceRecord when the callback from the producer client is received. This method is also called when a record is filtered by a transformation or when "errors.tolerance" is set to "all" and thus will never be ACK'd by a broker. In both cases metadata will be null.
SourceTasks are not required to implement this functionality; Kafka Connect will record offsets automatically. This hook is provided for systems that also need to store offsets internally in their own system.
The default implementation just calls commitRecord(SourceRecord), which is a nop by default. It is not necessary to implement both methods.
record - SourceRecord that was successfully sent via the producer, filtered by a transformation, or dropped on producer exceptionmetadata - RecordMetadata record metadata returned from the broker, or null if the record was filtered or if producer exceptions are ignoredInterruptedException