This is an automated email from the ASF dual-hosted git repository. sewen pushed a commit to branch release-1.14 in repository https://gitbox.apache.org/repos/asf/flink.git
commit 4aa457d75a83f929104ea1a4f7526ce13b06c475 Author: Stephan Ewen <se...@apache.org> AuthorDate: Mon Apr 19 18:05:01 2021 +0200 [FLINK-22357][core] Upgrades API stability of unified source API to @Public (= stable). --- .../main/java/org/apache/flink/api/connector/source/Boundedness.java | 4 ++-- .../main/java/org/apache/flink/api/connector/source/ReaderInfo.java | 4 ++-- .../main/java/org/apache/flink/api/connector/source/ReaderOutput.java | 4 ++-- .../src/main/java/org/apache/flink/api/connector/source/Source.java | 4 ++-- .../main/java/org/apache/flink/api/connector/source/SourceEvent.java | 4 ++-- .../main/java/org/apache/flink/api/connector/source/SourceOutput.java | 4 ++-- .../main/java/org/apache/flink/api/connector/source/SourceReader.java | 4 ++-- .../org/apache/flink/api/connector/source/SourceReaderContext.java | 4 ++-- .../main/java/org/apache/flink/api/connector/source/SourceSplit.java | 4 ++-- .../java/org/apache/flink/api/connector/source/SplitEnumerator.java | 4 ++-- .../org/apache/flink/api/connector/source/SplitEnumeratorContext.java | 4 ++-- .../java/org/apache/flink/api/connector/source/SplitsAssignment.java | 4 ++-- .../apache/flink/api/connector/source/lib/NumberSequenceSource.java | 2 ++ .../flink/api/connector/source/lib/util/IteratorSourceEnumerator.java | 2 ++ .../flink/api/connector/source/lib/util/IteratorSourceReader.java | 2 ++ .../flink/api/connector/source/lib/util/IteratorSourceSplit.java | 2 ++ 16 files changed, 32 insertions(+), 24 deletions(-) diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/Boundedness.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/Boundedness.java index 7c573eb..5bc0be7 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/Boundedness.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/Boundedness.java @@ -18,13 +18,13 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; /** * The boundedness of a stream. A stream could either be "bounded" (a stream with finite records) or * "unbounded" (a stream with infinite records). */ -@PublicEvolving +@Public public enum Boundedness { /** * A BOUNDED stream is a stream with finite records. diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderInfo.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderInfo.java index 87bd0f9..20e5b3f 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderInfo.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderInfo.java @@ -18,13 +18,13 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import java.io.Serializable; import java.util.Objects; /** A container class hosting the information of a {@link SourceReader}. */ -@PublicEvolving +@Public public final class ReaderInfo implements Serializable { private static final long serialVersionUID = 1L; diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderOutput.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderOutput.java index 2832cff..a9bd9d13 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderOutput.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/ReaderOutput.java @@ -18,7 +18,7 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import org.apache.flink.api.common.eventtime.TimestampAssigner; import org.apache.flink.api.common.eventtime.Watermark; @@ -38,7 +38,7 @@ import org.apache.flink.api.common.eventtime.Watermark; * ReaderOutput#createOutputForSplit(String)} method, using the Source Split's ID. Make sure to * release the output again once the source has finished processing that split. */ -@PublicEvolving +@Public public interface ReaderOutput<T> extends SourceOutput<T> { /** diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/Source.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/Source.java index 6e84d00..d186787 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/Source.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/Source.java @@ -18,7 +18,7 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import org.apache.flink.core.io.SimpleVersionedSerializer; import java.io.Serializable; @@ -31,7 +31,7 @@ import java.io.Serializable; * @param <SplitT> The type of splits handled by the source. * @param <EnumChkT> The type of the enumerator checkpoints. */ -@PublicEvolving +@Public public interface Source<T, SplitT extends SourceSplit, EnumChkT> extends Serializable { /** diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceEvent.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceEvent.java index cefaaaa7..31c0c85 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceEvent.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceEvent.java @@ -18,10 +18,10 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import java.io.Serializable; /** An base class for the events passed between the SourceReaders and Enumerators. */ -@PublicEvolving +@Public public interface SourceEvent extends Serializable {} diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceOutput.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceOutput.java index 67e5e51..560f071 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceOutput.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceOutput.java @@ -18,7 +18,7 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import org.apache.flink.api.common.eventtime.TimestampAssigner; import org.apache.flink.api.common.eventtime.WatermarkOutput; @@ -30,7 +30,7 @@ import org.apache.flink.api.common.eventtime.WatermarkOutput; * Splits</i>. That way, streams of events from different splits can be identified and treated * separately, for example for watermark generation, or event-time skew handling. */ -@PublicEvolving +@Public public interface SourceOutput<T> extends WatermarkOutput { /** diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReader.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReader.java index f7c17d4..aae78d2 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReader.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReader.java @@ -18,7 +18,7 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import org.apache.flink.api.common.state.CheckpointListener; import org.apache.flink.core.io.InputStatus; import org.apache.flink.metrics.Gauge; @@ -45,7 +45,7 @@ import java.util.concurrent.CompletableFuture; * @param <T> The type of the record emitted by this source reader. * @param <SplitT> The type of the the source splits. */ -@PublicEvolving +@Public public interface SourceReader<T, SplitT extends SourceSplit> extends AutoCloseable, CheckpointListener { diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReaderContext.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReaderContext.java index 7b7f877..304afdd 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReaderContext.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceReaderContext.java @@ -18,13 +18,13 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import org.apache.flink.configuration.Configuration; import org.apache.flink.metrics.groups.SourceReaderMetricGroup; import org.apache.flink.util.UserCodeClassLoader; /** The class that expose some context from runtime to the {@link SourceReader}. */ -@PublicEvolving +@Public public interface SourceReaderContext { /** @return The metric group this source belongs to. */ diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceSplit.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceSplit.java index e50337d..02576b0 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceSplit.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SourceSplit.java @@ -18,10 +18,10 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; /** An interface for all the Split types to extend. */ -@PublicEvolving +@Public public interface SourceSplit { /** diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumerator.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumerator.java index 2654671..dd7e19b 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumerator.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumerator.java @@ -18,7 +18,7 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import org.apache.flink.api.common.state.CheckpointListener; import javax.annotation.Nullable; @@ -30,7 +30,7 @@ import java.util.List; * A interface of a split enumerator responsible for the followings: 1. discover the splits for the * {@link SourceReader} to read. 2. assign the splits to the source reader. */ -@PublicEvolving +@Public public interface SplitEnumerator<SplitT extends SourceSplit, CheckpointT> extends AutoCloseable, CheckpointListener { diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumeratorContext.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumeratorContext.java index 0a7b0ef..897b681 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumeratorContext.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitEnumeratorContext.java @@ -18,7 +18,7 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import org.apache.flink.metrics.groups.SplitEnumeratorMetricGroup; import java.util.Map; @@ -33,7 +33,7 @@ import java.util.function.BiConsumer; * * @param <SplitT> the type of the splits. */ -@PublicEvolving +@Public public interface SplitEnumeratorContext<SplitT extends SourceSplit> { SplitEnumeratorMetricGroup metricGroup(); diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitsAssignment.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitsAssignment.java index 45f3216..a58a18a 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitsAssignment.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/SplitsAssignment.java @@ -18,7 +18,7 @@ package org.apache.flink.api.connector.source; -import org.apache.flink.annotation.PublicEvolving; +import org.apache.flink.annotation.Public; import java.util.Collections; import java.util.HashMap; @@ -31,7 +31,7 @@ import java.util.Map; * <p>The assignment is always incremental. In another word, splits in the assignment are simply * added to the existing assignment. */ -@PublicEvolving +@Public public final class SplitsAssignment<SplitT extends SourceSplit> { private final Map<Integer, List<SplitT>> assignment; diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/NumberSequenceSource.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/NumberSequenceSource.java index 73c2928..53e6a28 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/NumberSequenceSource.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/NumberSequenceSource.java @@ -18,6 +18,7 @@ package org.apache.flink.api.connector.source.lib; +import org.apache.flink.annotation.Public; import org.apache.flink.api.common.typeinfo.TypeInformation; import org.apache.flink.api.common.typeinfo.Types; import org.apache.flink.api.connector.source.Boundedness; @@ -57,6 +58,7 @@ import static org.apache.flink.util.Preconditions.checkNotNull; * long integer values), user may want to consider executing the application in a streaming manner, * because, despite the fact that the produced stream is bounded, the end bound is pretty far away. */ +@Public public class NumberSequenceSource implements Source< Long, diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceEnumerator.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceEnumerator.java index ff805c5..43ce317 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceEnumerator.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceEnumerator.java @@ -18,6 +18,7 @@ package org.apache.flink.api.connector.source.lib.util; +import org.apache.flink.annotation.Public; import org.apache.flink.api.connector.source.SplitEnumerator; import org.apache.flink.api.connector.source.SplitEnumeratorContext; @@ -36,6 +37,7 @@ import static org.apache.flink.util.Preconditions.checkNotNull; * * @param <SplitT> The type of the splits used by the source. */ +@Public public class IteratorSourceEnumerator<SplitT extends IteratorSourceSplit<?, ?>> implements SplitEnumerator<SplitT, Collection<SplitT>> { diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceReader.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceReader.java index 801b87b..d7a63c0 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceReader.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceReader.java @@ -18,6 +18,7 @@ package org.apache.flink.api.connector.source.lib.util; +import org.apache.flink.annotation.Public; import org.apache.flink.api.connector.source.ReaderOutput; import org.apache.flink.api.connector.source.SourceReader; import org.apache.flink.api.connector.source.SourceReaderContext; @@ -48,6 +49,7 @@ import static org.apache.flink.util.Preconditions.checkNotNull; * @param <SplitT> The concrete type of the {@code IteratorSourceSplit} that creates and converts * the iterator that produces this reader's elements. */ +@Public public class IteratorSourceReader< E, IterT extends Iterator<E>, SplitT extends IteratorSourceSplit<E, IterT>> implements SourceReader<E, SplitT> { diff --git a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceSplit.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceSplit.java index b475537..0b3d797 100644 --- a/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceSplit.java +++ b/flink-core/src/main/java/org/apache/flink/api/connector/source/lib/util/IteratorSourceSplit.java @@ -18,6 +18,7 @@ package org.apache.flink.api.connector.source.lib.util; +import org.apache.flink.annotation.Public; import org.apache.flink.api.connector.source.SourceSplit; import java.util.Iterator; @@ -28,6 +29,7 @@ import java.util.Iterator; * * @param <E> The type of the elements returned by the iterator. */ +@Public public interface IteratorSourceSplit<E, IterT extends Iterator<E>> extends SourceSplit { /** Gets the iterator over the elements of this split. */