@PublicEvolving public class SingleThreadFetcherManager<E,SplitT extends SourceSplit> extends SplitFetcherManager<E,SplitT>
This pattern is, for example, useful for connectors like File Readers, Apache Kafka Readers, etc. In the example of Kafka, there is a single thread that reads all splits (topic partitions) via the same client. In the example of the file source, there is a single thread that reads the files after another.
fetchers
Modifier and Type | Method and Description |
---|---|
void |
addSplits(List<SplitT> splitsToAdd) |
protected SplitFetcher<E,SplitT> |
getRunningFetcher() |
void |
removeSplits(List<SplitT> splitsToRemove) |
checkErrors, close, createSplitFetcher, getNumAliveFetchers, getQueue, maybeShutdownFinishedFetchers, pauseOrResumeSplits, startFetcher
@Deprecated public SingleThreadFetcherManager(FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> elementsQueue, Supplier<SplitReader<E,SplitT>> splitReaderSupplier)
SingleThreadFetcherManager(Supplier, Configuration)
instead.elementsQueue
- The queue that is used to hand over data from the I/O thread (the
fetchers) to the reader (which emits the records and book-keeps the state. This must be
the same queue instance that is also passed to the SourceReaderBase
.splitReaderSupplier
- The factory for the split reader that connects to the source
system.@Deprecated public SingleThreadFetcherManager(FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> elementsQueue, Supplier<SplitReader<E,SplitT>> splitReaderSupplier, Configuration configuration)
SingleThreadFetcherManager(Supplier, Configuration)
instead.elementsQueue
- The queue that is used to hand over data from the I/O thread (the
fetchers) to the reader (which emits the records and book-keeps the state. This must be
the same queue instance that is also passed to the SourceReaderBase
.splitReaderSupplier
- The factory for the split reader that connects to the source
system.configuration
- The configuration to create the fetcher manager.@VisibleForTesting @Deprecated public SingleThreadFetcherManager(FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> elementsQueue, Supplier<SplitReader<E,SplitT>> splitReaderSupplier, Configuration configuration, Consumer<Collection<String>> splitFinishedHook)
SingleThreadFetcherManager(Supplier, Configuration, Consumer)
instead.elementsQueue
- The queue that is used to hand over data from the I/O thread (the
fetchers) to the reader (which emits the records and book-keeps the state. This must be
the same queue instance that is also passed to the SourceReaderBase
.splitReaderSupplier
- The factory for the split reader that connects to the source
system.configuration
- The configuration to create the fetcher manager.splitFinishedHook
- Hook for handling finished splits in split fetcherspublic SingleThreadFetcherManager(Supplier<SplitReader<E,SplitT>> splitReaderSupplier)
splitReaderSupplier
- The factory for the split reader that connects to the source
system.public SingleThreadFetcherManager(Supplier<SplitReader<E,SplitT>> splitReaderSupplier, Configuration configuration)
splitReaderSupplier
- The factory for the split reader that connects to the source
system.configuration
- The configuration to create the fetcher manager.public SingleThreadFetcherManager(Supplier<SplitReader<E,SplitT>> splitReaderSupplier, Configuration configuration, Consumer<Collection<String>> splitFinishedHook)
splitReaderSupplier
- The factory for the split reader that connects to the source
system.configuration
- The configuration to create the fetcher manager.splitFinishedHook
- Hook for handling finished splits in split fetcherspublic void addSplits(List<SplitT> splitsToAdd)
addSplits
in class SplitFetcherManager<E,SplitT extends SourceSplit>
public void removeSplits(List<SplitT> splitsToRemove)
removeSplits
in class SplitFetcherManager<E,SplitT extends SourceSplit>
protected SplitFetcher<E,SplitT> getRunningFetcher()
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.