DBZ-555 Adding Denis Mikhaylov to COPYRIGHT.txt; formatting
This commit is contained in:
parent
048f1323cd
commit
1bc9dec6f5
@ -11,6 +11,7 @@ Christian Posta
|
||||
David Chen
|
||||
David Leibovic
|
||||
David Szabo
|
||||
Denis Mikhaylov
|
||||
Dennis Persson
|
||||
Duncan Sands
|
||||
Emrul Islam
|
||||
|
@ -60,7 +60,7 @@
|
||||
* Embedded connectors are designed to be submitted to an {@link Executor} or {@link ExecutorService} for execution by a single
|
||||
* thread, and a running connector can be stopped either by calling {@link #stop()} from another thread or by interrupting
|
||||
* the running thread (e.g., as is the case with {@link ExecutorService#shutdownNow()}).
|
||||
*
|
||||
*
|
||||
* @author Randall Hauch
|
||||
*/
|
||||
@ThreadSafe
|
||||
@ -92,7 +92,7 @@ public final class EmbeddedEngine implements Runnable {
|
||||
|
||||
/**
|
||||
* An optional field that specifies the file location for the {@link FileOffsetBackingStore}.
|
||||
*
|
||||
*
|
||||
* @see #OFFSET_STORAGE
|
||||
*/
|
||||
public static final Field OFFSET_STORAGE_FILE_FILENAME = Field.create(StandaloneConfig.OFFSET_STORAGE_FILE_FILENAME_CONFIG)
|
||||
@ -109,8 +109,7 @@ public final class EmbeddedEngine implements Runnable {
|
||||
public static final Field OFFSET_STORAGE_KAFKA_TOPIC = Field.create(DistributedConfig.OFFSET_STORAGE_TOPIC_CONFIG)
|
||||
.withDescription("The name of the Kafka topic where offsets are to be stored. "
|
||||
+ "Required with other properties when 'offset.storage' is set to the "
|
||||
+
|
||||
KafkaOffsetBackingStore.class.getName() + " class.")
|
||||
+ KafkaOffsetBackingStore.class.getName() + " class.")
|
||||
.withDefault("");
|
||||
|
||||
/**
|
||||
@ -118,26 +117,22 @@ public final class EmbeddedEngine implements Runnable {
|
||||
*
|
||||
* @see #OFFSET_STORAGE
|
||||
*/
|
||||
|
||||
public static final Field OFFSET_STORAGE_KAFKA_PARTITIONS = Field.create(DistributedConfig.OFFSET_STORAGE_PARTITIONS_CONFIG)
|
||||
.withType(ConfigDef.Type.INT)
|
||||
.withDescription("The number of partitions used when creating the offset storage topic. "
|
||||
+ "Required with other properties when 'offset.storage' is set to the "
|
||||
+
|
||||
KafkaOffsetBackingStore.class.getName() + " class.");
|
||||
+ KafkaOffsetBackingStore.class.getName() + " class.");
|
||||
|
||||
/**
|
||||
* An optional field that specifies the replication factor for the {@link KafkaOffsetBackingStore}.
|
||||
*
|
||||
* @see #OFFSET_STORAGE
|
||||
*/
|
||||
|
||||
public static final Field OFFSET_STORAGE_KAFKA_REPLICATION_FACTOR = Field.create(DistributedConfig.OFFSET_STORAGE_REPLICATION_FACTOR_CONFIG)
|
||||
.withType(ConfigDef.Type.SHORT)
|
||||
.withDescription("Replication factor used when creating the offset storage topic. "
|
||||
+ "Required with other properties when 'offset.storage' is set to the "
|
||||
+
|
||||
KafkaOffsetBackingStore.class.getName() + " class.");
|
||||
+ KafkaOffsetBackingStore.class.getName() + " class.");
|
||||
|
||||
/**
|
||||
* An optional advanced field that specifies the maximum amount of time that the embedded connector should wait
|
||||
@ -185,7 +180,7 @@ public final class EmbeddedEngine implements Runnable {
|
||||
public interface CompletionCallback {
|
||||
/**
|
||||
* Handle the completion of the embedded connector engine.
|
||||
*
|
||||
*
|
||||
* @param success {@code true} if the connector completed normally, or {@code false} if the connector produced an error
|
||||
* that prevented startup or premature termination.
|
||||
* @param message the completion message; never null
|
||||
@ -266,7 +261,7 @@ public void handle(boolean success, String message, Throwable error) {
|
||||
* or until the thread is {@linkplain Thread#interrupt interrupted}.
|
||||
* <p>
|
||||
* This method returns immediately if the connector has completed already.
|
||||
*
|
||||
*
|
||||
* @throws InterruptedException if the current thread is interrupted while waiting
|
||||
*/
|
||||
public void await() throws InterruptedException {
|
||||
@ -278,7 +273,7 @@ public void await() throws InterruptedException {
|
||||
* unless the thread is {@linkplain Thread#interrupt interrupted}, or the specified waiting time elapses.
|
||||
* <p>
|
||||
* This method returns immediately if the connector has completed already.
|
||||
*
|
||||
*
|
||||
* @param timeout the maximum time to wait
|
||||
* @param unit the time unit of the {@code timeout} argument
|
||||
* @return {@code true} if the completion was received, or {@code false} if the waiting time elapsed before the completion
|
||||
@ -291,7 +286,7 @@ public boolean await(long timeout, TimeUnit unit) throws InterruptedException {
|
||||
|
||||
/**
|
||||
* Determine if the connector has completed.
|
||||
*
|
||||
*
|
||||
* @return {@code true} if the connector has completed, or {@code false} if the connector is still running and this
|
||||
* callback has not yet been {@link #handle(boolean, String, Throwable) notified}
|
||||
*/
|
||||
@ -301,7 +296,7 @@ public boolean hasCompleted() {
|
||||
|
||||
/**
|
||||
* Get whether the connector completed normally.
|
||||
*
|
||||
*
|
||||
* @return {@code true} if the connector completed normally, or {@code false} if the connector produced an error that
|
||||
* prevented startup or premature termination (or the connector has not yet {@link #hasCompleted() completed})
|
||||
*/
|
||||
@ -311,7 +306,7 @@ public boolean success() {
|
||||
|
||||
/**
|
||||
* Get the completion message.
|
||||
*
|
||||
*
|
||||
* @return the completion message, or null if the connector has not yet {@link #hasCompleted() completed}
|
||||
*/
|
||||
public String message() {
|
||||
@ -320,7 +315,7 @@ public String message() {
|
||||
|
||||
/**
|
||||
* Get the completion error, if there is one.
|
||||
*
|
||||
*
|
||||
* @return the completion error, or null if there is no error or connector has not yet {@link #hasCompleted() completed}
|
||||
*/
|
||||
public Throwable error() {
|
||||
@ -329,7 +324,7 @@ public Throwable error() {
|
||||
|
||||
/**
|
||||
* Determine if there is a completion error.
|
||||
*
|
||||
*
|
||||
* @return {@code true} if there is a {@link #error completion error}, or {@code false} if there is no error or
|
||||
* the connector has not yet {@link #hasCompleted() completed}
|
||||
*/
|
||||
@ -346,7 +341,7 @@ public static interface Builder {
|
||||
/**
|
||||
* Call the specified function for every {@link SourceRecord data change event} read from the source database.
|
||||
* This method must be called with a non-null consumer.
|
||||
*
|
||||
*
|
||||
* @param consumer the consumer function
|
||||
* @return this builder object so methods can be chained together; never null
|
||||
*/
|
||||
@ -354,7 +349,7 @@ public static interface Builder {
|
||||
|
||||
/**
|
||||
* Use the specified configuration for the connector. The configuration is assumed to already be valid.
|
||||
*
|
||||
*
|
||||
* @param config the configuration
|
||||
* @return this builder object so methods can be chained together; never null
|
||||
*/
|
||||
@ -363,7 +358,7 @@ public static interface Builder {
|
||||
/**
|
||||
* Use the specified class loader to find all necessary classes. Passing <code>null</code> or not calling this method
|
||||
* results in the connector using this class's class loader.
|
||||
*
|
||||
*
|
||||
* @param classLoader the class loader
|
||||
* @return this builder object so methods can be chained together; never null
|
||||
*/
|
||||
@ -372,7 +367,7 @@ public static interface Builder {
|
||||
/**
|
||||
* Use the specified clock when needing to determine the current time. Passing <code>null</code> or not calling this
|
||||
* method results in the connector using the {@link Clock#system() system clock}.
|
||||
*
|
||||
*
|
||||
* @param clock the clock
|
||||
* @return this builder object so methods can be chained together; never null
|
||||
*/
|
||||
@ -380,7 +375,7 @@ public static interface Builder {
|
||||
|
||||
/**
|
||||
* When the engine's {@link EmbeddedEngine#run()} method completes, call the supplied function with the results.
|
||||
*
|
||||
*
|
||||
* @param completionCallback the callback function; may be null if errors should be written to the log
|
||||
* @return this builder object so methods can be chained together; never null
|
||||
*/
|
||||
@ -389,7 +384,7 @@ public static interface Builder {
|
||||
/**
|
||||
* During the engine's {@link EmbeddedEngine#run()} method, call the supplied the supplied function at different
|
||||
* stages according to the completion state of each component running within the engine (connectors, tasks etc)
|
||||
*
|
||||
*
|
||||
* @param connectorCallback the callback function; may be null
|
||||
* @return this builder object so methods can be chained together; never null
|
||||
*/
|
||||
@ -397,7 +392,7 @@ public static interface Builder {
|
||||
|
||||
/**
|
||||
* Build a new connector with the information previously supplied to this builder.
|
||||
*
|
||||
*
|
||||
* @return the embedded connector; never null
|
||||
* @throws IllegalArgumentException if a {@link #using(Configuration) configuration} or {@link #notifying(Consumer)
|
||||
* consumer function} were not supplied before this method is called
|
||||
@ -407,7 +402,7 @@ public static interface Builder {
|
||||
|
||||
/**
|
||||
* Obtain a new {@link Builder} instance that can be used to construct runnable {@link EmbeddedEngine} instances.
|
||||
*
|
||||
*
|
||||
* @return the new builder; never null
|
||||
*/
|
||||
public static Builder create() {
|
||||
@ -518,7 +513,7 @@ private EmbeddedEngine(Configuration config, ClassLoader classLoader, Clock cloc
|
||||
|
||||
/**
|
||||
* Determine if this embedded connector is currently running.
|
||||
*
|
||||
*
|
||||
* @return {@code true} if running, or {@code false} otherwise
|
||||
*/
|
||||
public boolean isRunning() {
|
||||
@ -831,7 +826,7 @@ protected void completedFlush(Throwable error, Void result) {
|
||||
/**
|
||||
* Stop the execution of this embedded connector. This method does not block until the connector is stopped; use
|
||||
* {@link #await(long, TimeUnit)} for this purpose.
|
||||
*
|
||||
*
|
||||
* @return {@code true} if the connector was {@link #run() running} and will eventually stop, or {@code false} if it was not
|
||||
* running when this method is called
|
||||
* @see #await(long, TimeUnit)
|
||||
@ -853,7 +848,7 @@ public boolean stop() {
|
||||
* Wait for the connector to complete processing. If the processor is not running, this method returns immediately; however,
|
||||
* if the processor is {@link #stop() stopped} and restarted before this method is called, this method will return only
|
||||
* when it completes the second time.
|
||||
*
|
||||
*
|
||||
* @param timeout the maximum amount of time to wait before returning
|
||||
* @param unit the unit of time; may not be null
|
||||
* @return {@code true} if the connector completed within the timeout (or was not running), or {@code false} if it is still
|
||||
|
Loading…
Reference in New Issue
Block a user