@PublicEvolving public abstract class SplitFetcherManager<E,SplitT extends SourceSplit> extends Object
SplitFetcher
and manage the life cycles of them.
This class works with the SourceReaderBase
.
The split fetcher manager could be used to support different threading models by implementing
the addSplits(List)
method differently. For example, a single thread split fetcher
manager would only start a single fetcher and assign all the splits to it. A one-thread-per-split
fetcher may spawn a new thread every time a new split is assigned.
Modifier and Type | Field and Description |
---|---|
protected Map<Integer,SplitFetcher<E,SplitT>> |
fetchers
A map keeping track of all the split fetchers.
|
Constructor and Description |
---|
SplitFetcherManager(FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> elementsQueue,
Supplier<SplitReader<E,SplitT>> splitReaderFactory,
Configuration configuration)
Deprecated.
Please use
SplitFetcherManager(Supplier, Configuration) instead. |
SplitFetcherManager(FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> elementsQueue,
Supplier<SplitReader<E,SplitT>> splitReaderFactory,
Configuration configuration,
Consumer<Collection<String>> splitFinishedHook)
Deprecated.
Please use
SplitFetcherManager(Supplier, Configuration, Consumer)
instead. |
SplitFetcherManager(Supplier<SplitReader<E,SplitT>> splitReaderFactory,
Configuration configuration)
Create a split fetcher manager.
|
SplitFetcherManager(Supplier<SplitReader<E,SplitT>> splitReaderFactory,
Configuration configuration,
Consumer<Collection<String>> splitFinishedHook)
Create a split fetcher manager.
|
Modifier and Type | Method and Description |
---|---|
abstract void |
addSplits(List<SplitT> splitsToAdd) |
void |
checkErrors() |
void |
close(long timeoutMs)
Close the split fetcher manager.
|
protected SplitFetcher<E,SplitT> |
createSplitFetcher()
Synchronize method to ensure no fetcher is created after the split fetcher manager has
closed.
|
int |
getNumAliveFetchers() |
FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> |
getQueue()
Return the queue contains data produced by split fetchers.This method is Internal and only
used in
SourceReaderBase . |
boolean |
maybeShutdownFinishedFetchers()
Check and shutdown the fetchers that have completed their work.
|
void |
pauseOrResumeSplits(Collection<String> splitIdsToPause,
Collection<String> splitIdsToResume) |
abstract void |
removeSplits(List<SplitT> splitsToRemove) |
protected void |
startFetcher(SplitFetcher<E,SplitT> fetcher) |
protected final Map<Integer,SplitFetcher<E,SplitT extends SourceSplit>> fetchers
@Deprecated public SplitFetcherManager(FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> elementsQueue, Supplier<SplitReader<E,SplitT>> splitReaderFactory, Configuration configuration)
SplitFetcherManager(Supplier, Configuration)
instead.elementsQueue
- the queue that split readers will put elements into.splitReaderFactory
- a supplier that could be used to create split readers.configuration
- the configuration of this fetcher manager.@Deprecated @VisibleForTesting public SplitFetcherManager(FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> elementsQueue, Supplier<SplitReader<E,SplitT>> splitReaderFactory, Configuration configuration, Consumer<Collection<String>> splitFinishedHook)
SplitFetcherManager(Supplier, Configuration, Consumer)
instead.elementsQueue
- the queue that split readers will put elements into.splitReaderFactory
- a supplier that could be used to create split readers.configuration
- the configuration of this fetcher manager.splitFinishedHook
- Hook for handling finished splits in split fetchers.public SplitFetcherManager(Supplier<SplitReader<E,SplitT>> splitReaderFactory, Configuration configuration)
splitReaderFactory
- a supplier that could be used to create split readers.configuration
- the configuration of this fetcher manager.public SplitFetcherManager(Supplier<SplitReader<E,SplitT>> splitReaderFactory, Configuration configuration, Consumer<Collection<String>> splitFinishedHook)
splitReaderFactory
- a supplier that could be used to create split readers.configuration
- the configuration of this fetcher manager.splitFinishedHook
- Hook for handling finished splits in split fetchers.public void pauseOrResumeSplits(Collection<String> splitIdsToPause, Collection<String> splitIdsToResume)
protected void startFetcher(SplitFetcher<E,SplitT> fetcher)
protected SplitFetcher<E,SplitT> createSplitFetcher()
IllegalStateException
- if the split fetcher manager has closed.public boolean maybeShutdownFinishedFetchers()
@Internal public FutureCompletingBlockingQueue<RecordsWithSplitIds<E>> getQueue()
SourceReaderBase
.public void close(long timeoutMs) throws Exception
timeoutMs
- the max time in milliseconds to wait.Exception
- when failed to close the split fetcher manager.public void checkErrors()
@VisibleForTesting public int getNumAliveFetchers()
Copyright © 2014–2024 The Apache Software Foundation. All rights reserved.