Skip to content

Commit

Permalink
[SPARK-42075][DSTREAM] Deprecate DStream API
Browse files Browse the repository at this point in the history
### What changes were proposed in this pull request?
Deprecate the DStream interface by deprecating the StreamingContext api.

### Why are the changes needed?
DStream is not longer maintained. Encourage user to migrate to structured streaming.

### Does this PR introduce _any_ user-facing change?
Yes, user that use DStream will see deprecation warning.

### How was this patch tested?
No functional change

Closes apache#39647 from chaoqin-li1123/deprecate_dstream.

Authored-by: Chaoqin Li <chaoqin.li@databricks.com>
Signed-off-by: Jungtaek Lim <kabhwan.opensource@gmail.com>
  • Loading branch information
chaoqin-li1123 authored and HeartSaVioR committed Jan 20, 2023
1 parent db06f3e commit ec424c5
Show file tree
Hide file tree
Showing 3 changed files with 32 additions and 1 deletion.
12 changes: 12 additions & 0 deletions python/pyspark/streaming/context.py
Original file line number Diff line number Diff line change
Expand Up @@ -26,6 +26,8 @@
from pyspark.streaming.listener import StreamingListener
from pyspark.streaming.util import TransformFunction, TransformFunctionSerializer

import warnings

__all__ = ["StreamingContext"]

T = TypeVar("T")
Expand All @@ -41,6 +43,12 @@ class StreamingContext:
respectively. `context.awaitTermination()` allows the current thread
to wait for the termination of the context by `stop()` or by an exception.
.. deprecated:: Spark 3.4.0
This is deprecated as of Spark 3.4.0.
There are no longer updates to DStream and it's a legacy project.
There is a newer and easier to use streaming engine in Spark called Structured Streaming.
You should use Spark Structured Streaming for your streaming applications.
Parameters
----------
sparkContext : :class:`SparkContext`
Expand All @@ -61,6 +69,10 @@ def __init__(
batchDuration: Optional[int] = None,
jssc: Optional[JavaObject] = None,
):
warnings.warn(
"DStream is deprecated as of Spark 3.4.0. Migrate to Structured Streaming.",
FutureWarning,
)
self._sc = sparkContext
self._jvm = self._sc._jvm
self._jssc = jssc or self._initialize_context(self._sc, batchDuration)
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -60,7 +60,13 @@ import org.apache.spark.util.{CallSite, ShutdownHookManager, ThreadUtils, Utils}
* using `context.start()` and `context.stop()`, respectively.
* `context.awaitTermination()` allows the current thread to wait for the termination
* of the context by `stop()` or by an exception.
* @deprecated This is deprecated as of Spark 3.4.0.
* There are no longer updates to DStream and it's a legacy project.
* There is a newer and easier to use streaming engine
* in Spark called Structured Streaming.
* You should use Spark Structured Streaming for your streaming applications.
*/
@deprecated("DStream is deprecated. Migrate to Structured Streaming.", "Spark 3.4.0")
class StreamingContext private[streaming] (
_sc: SparkContext,
_cp: Checkpoint,
Expand Down Expand Up @@ -740,13 +746,20 @@ class StreamingContext private[streaming] (
/**
* StreamingContext object contains a number of utility functions related to the
* StreamingContext class.
*
* @deprecated This is deprecated as of Spark 3.4.0.
* There are no longer updates to DStream and it's a legacy project.
* There is a newer and easier to use streaming engine
* in Spark called Structured Streaming.
* You should use Spark Structured Streaming for your streaming applications.
*/

@deprecated("DStream is deprecated. Migrate to Structured Streaming.", "Spark 3.4.0")
object StreamingContext extends Logging {

/**
* Lock that guards activation of a StreamingContext as well as access to the singleton active
* StreamingContext in getActiveOrCreate().
*
*/
private val ACTIVATION_LOCK = new Object()

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -50,7 +50,13 @@ import org.apache.spark.streaming.scheduler.StreamingListener
* computation can be started and stopped using `context.start()` and `context.stop()`,
* respectively. `context.awaitTermination()` allows the current thread to wait for the
* termination of a context by `stop()` or by an exception.
* @deprecated This is deprecated as of Spark 3.4.0.
* There are no longer updates to DStream and it's a legacy project.
* There is a newer and easier to use streaming engine
* in Spark called Structured Streaming.
* You should use Spark Structured Streaming for your streaming applications.
*/
@deprecated("DStream is deprecated. Migrate to Structured Streaming.", "Spark 3.4.0")
class JavaStreamingContext(val ssc: StreamingContext) extends Closeable {

/**
Expand Down

0 comments on commit ec424c5

Please sign in to comment.