Skip to content

Commit 1ed4049

Browse files
committed
[FLINK-14068][streaming] Remove unrelated scala APIs
1 parent 6f7ed6f commit 1ed4049

File tree

3 files changed

+8
-234
lines changed

3 files changed

+8
-234
lines changed

flink-runtime/src/main/java/org/apache/flink/streaming/api/windowing/time/Time.java

Lines changed: 0 additions & 139 deletions
This file was deleted.

flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/DataStream.scala

Lines changed: 2 additions & 50 deletions
Original file line numberDiff line numberDiff line change
@@ -38,10 +38,11 @@ import org.apache.flink.streaming.api.functions.sink.SinkFunction
3838
import org.apache.flink.streaming.api.functions.timestamps.{AscendingTimestampExtractor, BoundedOutOfOrdernessTimestampExtractor}
3939
import org.apache.flink.streaming.api.operators.OneInputStreamOperator
4040
import org.apache.flink.streaming.api.windowing.assigners._
41-
import org.apache.flink.streaming.api.windowing.time.Time
4241
import org.apache.flink.streaming.api.windowing.windows.{GlobalWindow, TimeWindow, Window}
4342
import org.apache.flink.util.{CloseableIterator, Collector}
4443

44+
import java.time.Duration
45+
4546
import scala.collection.JavaConverters._
4647

4748
/**
@@ -740,55 +741,6 @@ class DataStream[T](stream: JavaStream[T]) {
740741
filter(filterFun)
741742
}
742743

743-
/**
744-
* Windows this DataStream into tumbling time windows.
745-
*
746-
* This is a shortcut for either `.window(TumblingEventTimeWindows.of(size))` or
747-
* `.window(TumblingProcessingTimeWindows.of(size))` depending on the time characteristic set
748-
* using [[StreamExecutionEnvironment.setStreamTimeCharacteristic]].
749-
*
750-
* Note: This operation can be inherently non-parallel since all elements have to pass through the
751-
* same operator instance. (Only for special cases, such as aligned time windows is it possible to
752-
* perform this operation in parallel).
753-
*
754-
* @param size
755-
* The size of the window.
756-
*
757-
* @deprecated
758-
* Please use [[windowAll()]] with either [[TumblingEventTimeWindows]] or
759-
* [[TumblingProcessingTimeWindows]]. For more information, see the deprecation notice on
760-
* [[org.apache.flink.streaming.api.TimeCharacteristic]].
761-
*/
762-
@deprecated
763-
def timeWindowAll(size: Time): AllWindowedStream[T, TimeWindow] = {
764-
new AllWindowedStream(javaStream.timeWindowAll(size))
765-
}
766-
767-
/**
768-
* Windows this DataStream into sliding time windows.
769-
*
770-
* This is a shortcut for either `.window(SlidingEventTimeWindows.of(size, slide))` or
771-
* `.window(SlidingProcessingTimeWindows.of(size, slide))` depending on the time characteristic
772-
* set using [[StreamExecutionEnvironment.setStreamTimeCharacteristic]].
773-
*
774-
* Note: This operation can be inherently non-parallel since all elements have to pass through the
775-
* same operator instance. (Only for special cases, such as aligned time windows is it possible to
776-
* perform this operation in parallel).
777-
*
778-
* @param size
779-
* The size of the window.
780-
*
781-
* @deprecated
782-
* Please use [[windowAll()]] with either [[SlidingEventTimeWindows]] or
783-
* [[SlidingProcessingTimeWindows]]. For more information, see the deprecation notice on
784-
* [[org.apache.flink.streaming.api.TimeCharacteristic]].
785-
*/
786-
@deprecated
787-
def timeWindowAll(size: Time, slide: Time): AllWindowedStream[T, TimeWindow] = {
788-
new AllWindowedStream(javaStream.timeWindowAll(size, slide))
789-
790-
}
791-
792744
/**
793745
* Windows this [[DataStream]] into sliding count windows.
794746
*

flink-streaming-scala/src/main/scala/org/apache/flink/streaming/api/scala/KeyedStream.scala

Lines changed: 6 additions & 45 deletions
Original file line numberDiff line numberDiff line change
@@ -30,10 +30,11 @@ import org.apache.flink.streaming.api.functions.co.ProcessJoinFunction
3030
import org.apache.flink.streaming.api.functions.query.{QueryableAppendingStateOperator, QueryableValueStateOperator}
3131
import org.apache.flink.streaming.api.scala.function.StatefulFunction
3232
import org.apache.flink.streaming.api.windowing.assigners._
33-
import org.apache.flink.streaming.api.windowing.time.Time
34-
import org.apache.flink.streaming.api.windowing.windows.{GlobalWindow, TimeWindow, Window}
33+
import org.apache.flink.streaming.api.windowing.windows.{GlobalWindow, Window}
3534
import org.apache.flink.util.Collector
3635

36+
import java.time.Duration
37+
3738
/**
3839
* @deprecated
3940
* All Flink Scala APIs are deprecated and will be removed in a future Flink major version. You
@@ -162,9 +163,9 @@ class KeyedStream[T, K](javaStream: KeyedJavaStream[T, K]) extends DataStream[T]
162163
* The upper bound. Needs to be bigger than or equal to the lowerBound
163164
*/
164165
@PublicEvolving
165-
def between(lowerBound: Time, upperBound: Time): IntervalJoined[IN1, IN2, KEY] = {
166-
val lowerMillis = lowerBound.toMilliseconds
167-
val upperMillis = upperBound.toMilliseconds
166+
def between(lowerBound: Duration, upperBound: Duration): IntervalJoined[IN1, IN2, KEY] = {
167+
val lowerMillis = lowerBound.toMillis
168+
val upperMillis = upperBound.toMillis
168169
new IntervalJoined[IN1, IN2, KEY](streamOne, streamTwo, lowerMillis, upperMillis)
169170
}
170171
}
@@ -236,46 +237,6 @@ class KeyedStream[T, K](javaStream: KeyedJavaStream[T, K]) extends DataStream[T]
236237
// Windowing
237238
// ------------------------------------------------------------------------
238239

239-
/**
240-
* Windows this [[KeyedStream]] into tumbling time windows.
241-
*
242-
* This is a shortcut for either `.window(TumblingEventTimeWindows.of(size))` or
243-
* `.window(TumblingProcessingTimeWindows.of(size))` depending on the time characteristic set
244-
* using [[StreamExecutionEnvironment.setStreamTimeCharacteristic()]]
245-
*
246-
* @param size
247-
* The size of the window.
248-
*
249-
* @deprecated
250-
* Please use [[window()]] with either [[TumblingEventTimeWindows]] or
251-
* [[TumblingProcessingTimeWindows]]. For more information, see the deprecation notice on
252-
* [[org.apache.flink.streaming.api.TimeCharacteristic]].
253-
*/
254-
@deprecated
255-
def timeWindow(size: Time): WindowedStream[T, K, TimeWindow] = {
256-
new WindowedStream(javaStream.timeWindow(size))
257-
}
258-
259-
/**
260-
* Windows this [[KeyedStream]] into sliding time windows.
261-
*
262-
* This is a shortcut for either `.window(SlidingEventTimeWindows.of(size))` or
263-
* `.window(SlidingProcessingTimeWindows.of(size))` depending on the time characteristic set using
264-
* [[StreamExecutionEnvironment.setStreamTimeCharacteristic()]]
265-
*
266-
* @param size
267-
* The size of the window.
268-
*
269-
* @deprecated
270-
* Please use [[window()]] with either [[SlidingEventTimeWindows]] or
271-
* [[SlidingProcessingTimeWindows]]. For more information, see the deprecation notice on
272-
* [[org.apache.flink.streaming.api.TimeCharacteristic]].
273-
*/
274-
@deprecated
275-
def timeWindow(size: Time, slide: Time): WindowedStream[T, K, TimeWindow] = {
276-
new WindowedStream(javaStream.timeWindow(size, slide))
277-
}
278-
279240
/**
280241
* Windows this [[KeyedStream]] into sliding count windows.
281242
*

0 commit comments

Comments
 (0)