You are viewing a plain text version of this content. The canonical link for it is here.
Posted to commits@flink.apache.org by se...@apache.org on 2020/05/27 20:01:01 UTC

[flink] 05/16: [FLINK-17897][core] Classify FLIP-27 source API to @Experimental / @PublicEvolving

This is an automated email from the ASF dual-hosted git repository.

sewen pushed a commit to branch master
in repository https://gitbox.apache.org/repos/asf/flink.git

commit c6b5a7a2d7f6b405b6c0cbb0a6f6364eb87f2937
Author: Stephan Ewen <se...@apache.org>
AuthorDate: Sun May 24 00:36:30 2020 +0200

    [FLINK-17897][core] Classify FLIP-27 source API to @Experimental / @PublicEvolving
---
 .../main/java/org/apache/flink/api/connector/source/Boundedness.java | 4 ++--
 .../main/java/org/apache/flink/api/connector/source/ReaderInfo.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 ++--
 .../java/org/apache/flink/api/connector/source/SourceOutput.java     | 4 ++--
 .../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 ++--
 .../apache/flink/api/connector/source/SplitEnumeratorContext.java    | 4 ++--
 .../java/org/apache/flink/api/connector/source/SplitsAssignment.java | 4 ++--
 .../flink/streaming/api/environment/StreamExecutionEnvironment.java  | 5 +++--
 .../flink/streaming/api/scala/StreamExecutionEnvironment.scala       | 3 ++-
 13 files changed, 27 insertions(+), 25 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 5b5c4a6..d09318e 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.Public;
+import org.apache.flink.annotation.PublicEvolving;
 
 /**
  * The boundedness of a stream. A stream could either be "bounded" (a stream with finite records) or
  * "unbounded" (a stream with infinite records).
  */
-@Public
+@PublicEvolving
 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 f4a8f8d..f8535e59 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,7 +18,7 @@
 
 package org.apache.flink.api.connector.source;
 
-import org.apache.flink.annotation.Public;
+import org.apache.flink.annotation.PublicEvolving;
 
 import java.io.Serializable;
 import java.util.Objects;
@@ -26,7 +26,7 @@ import java.util.Objects;
 /**
  * A container class hosting the information of a {@link SourceReader}.
  */
-@Public
+@PublicEvolving
 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/Source.java b/flink-core/src/main/java/org/apache/flink/api/connector/source/Source.java
index 1b8a1ca..53fc065 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.Public;
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.core.io.SimpleVersionedSerializer;
 
 import java.io.IOException;
@@ -33,7 +33,7 @@ import java.io.Serializable;
  * @param <SplitT>   The type of splits handled by the source.
  * @param <EnumChkT> The type of the enumerator checkpoints.
  */
-@Public
+@PublicEvolving
 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 2377bb7..c21aaba 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,14 +18,14 @@
 
 package org.apache.flink.api.connector.source;
 
-import org.apache.flink.annotation.Public;
+import org.apache.flink.annotation.PublicEvolving;
 
 import java.io.Serializable;
 
 /**
  * An base class for the events passed between the SourceReaders and Enumerators.
  */
-@Public
+@PublicEvolving
 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 3896b15..ccf560b 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,14 +18,14 @@
 
 package org.apache.flink.api.connector.source;
 
-import org.apache.flink.annotation.Public;
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.api.common.eventtime.WatermarkOutput;
 
 /**
  * The interface provided by Flink task to the {@link SourceReader} to emit records
  * to downstream operators for message processing.
  */
-@Public
+@PublicEvolving
 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 66a1229..6a26d70 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.Public;
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.core.io.InputStatus;
 
 import java.util.List;
@@ -31,7 +31,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.
  */
-@Public
+@PublicEvolving
 public interface SourceReader<T, SplitT extends SourceSplit> extends AutoCloseable {
 
 	/**
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 2936649..ead9ae2 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.Public;
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.metrics.MetricGroup;
 
 /**
  * The class that expose some context from runtime to the {@link SourceReader}.
  */
-@Public
+@PublicEvolving
 public interface SourceReaderContext {
 
 	/**
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 d317f0e..7f2040a 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,12 +18,12 @@
 
 package org.apache.flink.api.connector.source;
 
-import org.apache.flink.annotation.Public;
+import org.apache.flink.annotation.PublicEvolving;
 
 /**
  * An interface for all the Split types to extend.
  */
-@Public
+@PublicEvolving
 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 16b3938..517a492 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.Public;
+import org.apache.flink.annotation.PublicEvolving;
 
 import java.io.IOException;
 import java.util.List;
@@ -28,7 +28,7 @@ import java.util.List;
  * 1. discover the splits for the {@link SourceReader} to read.
  * 2. assign the splits to the source reader.
  */
-@Public
+@PublicEvolving
 public interface SplitEnumerator<SplitT extends SourceSplit, CheckpointT> extends AutoCloseable {
 
 	/**
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 db6ccad..33f4a4c 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.Public;
+import org.apache.flink.annotation.PublicEvolving;
 import org.apache.flink.metrics.MetricGroup;
 
 import java.util.Map;
@@ -34,7 +34,7 @@ import java.util.function.BiConsumer;
  *
  * @param <SplitT> the type of the splits.
  */
-@Public
+@PublicEvolving
 public interface SplitEnumeratorContext<SplitT extends SourceSplit> {
 
 	MetricGroup 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 aad3577..33d8a13 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.Public;
+import org.apache.flink.annotation.PublicEvolving;
 
 import java.util.List;
 import java.util.Map;
@@ -29,7 +29,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.
  */
-@Public
+@PublicEvolving
 public final class SplitsAssignment<SplitT extends SourceSplit> {
 	private final Map<Integer, List<SplitT>> assignment;
 
diff --git a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java
index c8d038a..52204cf 100644
--- a/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java
+++ b/flink-streaming-java/src/main/java/org/apache/flink/streaming/api/environment/StreamExecutionEnvironment.java
@@ -17,6 +17,7 @@
 
 package org.apache.flink.streaming.api.environment;
 
+import org.apache.flink.annotation.Experimental;
 import org.apache.flink.annotation.Internal;
 import org.apache.flink.annotation.Public;
 import org.apache.flink.annotation.PublicEvolving;
@@ -1612,7 +1613,7 @@ public class StreamExecutionEnvironment {
 	 * 		type of the returned stream
 	 * @return the data stream constructed
 	 */
-	@PublicEvolving
+	@Experimental
 	public <OUT> DataStreamSource<OUT> continuousSource(Source<OUT, ?, ?> source, String sourceName) {
 		return continuousSource(source, sourceName, null);
 	}
@@ -1630,7 +1631,7 @@ public class StreamExecutionEnvironment {
 	 * 		the user defined type information for the stream
 	 * @return the data stream constructed
 	 */
-	@PublicEvolving
+	@Experimental
 	public <OUT> DataStreamSource<OUT> continuousSource(
 			Source<OUT, ?, ?> source,
 			String sourceName,
diff --git a/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala b/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
index 37bb4f9..509ab65 100644
--- a/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
+++ b/flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/StreamExecutionEnvironment.scala
@@ -19,7 +19,7 @@
 package org.apache.flink.streaming.api.scala
 
 import com.esotericsoftware.kryo.Serializer
-import org.apache.flink.annotation.{Internal, Public, PublicEvolving}
+import org.apache.flink.annotation.{Experimental, Internal, Public, PublicEvolving}
 import org.apache.flink.api.common.io.{FileInputFormat, FilePathFilter, InputFormat}
 import org.apache.flink.api.common.restartstrategy.RestartStrategies.RestartStrategyConfiguration
 import org.apache.flink.api.common.typeinfo.TypeInformation
@@ -664,6 +664,7 @@ class StreamExecutionEnvironment(javaEnv: JavaEnv) {
   /**
     * Create a DataStream using a [[Source]].
     */
+  @Experimental
   def continuousSource[T: TypeInformation](
       source: Source[T, _ <: SourceSplit, _],
       sourceName: String): Unit = {