-
Notifications
You must be signed in to change notification settings - Fork 28.9k
[SPARK-24396] [SS] [PYSPARK] Add Structured Streaming ForeachWriter for python #21477
New issue
Have a question about this project? Sign up for a free GitHub account to open an issue and contact its maintainers and the community.
By clicking “Sign up for GitHub”, you agree to our terms of service and privacy statement. We’ll occasionally send you account related emails.
Already on GitHub? Sign in to your account
Changes from all commits
701a455
0920260
f40dff6
d1cd933
8e30e8d
ecf3d88
d081110
1ab612f
File filter
Filter by extension
Conversations
Jump to
Diff view
Diff view
There are no files selected for viewing
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -843,6 +843,168 @@ def trigger(self, processingTime=None, once=None, continuous=None): | |
| self._jwrite = self._jwrite.trigger(jTrigger) | ||
| return self | ||
|
|
||
| @since(2.4) | ||
| def foreach(self, f): | ||
| """ | ||
| Sets the output of the streaming query to be processed using the provided writer ``f``. | ||
| This is often used to write the output of a streaming query to arbitrary storage systems. | ||
| The processing logic can be specified in two ways. | ||
|
|
||
| #. A **function** that takes a row as input. | ||
| This is a simple way to express your processing logic. Note that this does | ||
| not allow you to deduplicate generated data when failures cause reprocessing of | ||
| some input data. That would require you to specify the processing logic in the next | ||
| way. | ||
|
|
||
| #. An **object** with a ``process`` method and optional ``open`` and ``close`` methods. | ||
|
There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. @tdas, wouldn't we better just have There was a problem hiding this comment. Choose a reason for hiding this commentThe reason will be displayed to describe this comment to others. Learn more. I discussed this with @marmbrus . If there is a ForeachWriter class in python, then uses will have to additionally import it. That's just another overhead that can be avoided by just allowing any class with the appropriate methods. One less step for python users. |
||
| The object can have the following methods. | ||
|
|
||
| * ``open(partition_id, epoch_id)``: *Optional* method that initializes the processing | ||
| (for example, open a connection, start a transaction, etc). Additionally, you can | ||
| use the `partition_id` and `epoch_id` to deduplicate regenerated data | ||
| (discussed later). | ||
|
|
||
| * ``process(row)``: *Non-optional* method that processes each :class:`Row`. | ||
|
|
||
| * ``close(error)``: *Optional* method that finalizes and cleans up (for example, | ||
| close connection, commit transaction, etc.) after all rows have been processed. | ||
|
|
||
| The object will be used by Spark in the following way. | ||
|
|
||
| * A single copy of this object is responsible of all the data generated by a | ||
| single task in a query. In other words, one instance is responsible for | ||
| processing one partition of the data generated in a distributed manner. | ||
|
|
||
| * This object must be serializable because each task will get a fresh | ||
| serialized-deserialized copy of the provided object. Hence, it is strongly | ||
| recommended that any initialization for writing data (e.g. opening a | ||
| connection or starting a transaction) is done after the `open(...)` | ||
| method has been called, which signifies that the task is ready to generate data. | ||
|
|
||
| * The lifecycle of the methods are as follows. | ||
|
|
||
| For each partition with ``partition_id``: | ||
|
|
||
| ... For each batch/epoch of streaming data with ``epoch_id``: | ||
|
|
||
| ....... Method ``open(partitionId, epochId)`` is called. | ||
|
|
||
| ....... If ``open(...)`` returns true, for each row in the partition and | ||
| batch/epoch, method ``process(row)`` is called. | ||
|
|
||
| ....... Method ``close(errorOrNull)`` is called with error (if any) seen while | ||
| processing rows. | ||
|
|
||
| Important points to note: | ||
|
|
||
| * The `partitionId` and `epochId` can be used to deduplicate generated data when | ||
| failures cause reprocessing of some input data. This depends on the execution | ||
| mode of the query. If the streaming query is being executed in the micro-batch | ||
| mode, then every partition represented by a unique tuple (partition_id, epoch_id) | ||
| is guaranteed to have the same data. Hence, (partition_id, epoch_id) can be used | ||
| to deduplicate and/or transactionally commit data and achieve exactly-once | ||
| guarantees. However, if the streaming query is being executed in the continuous | ||
| mode, then this guarantee does not hold and therefore should not be used for | ||
| deduplication. | ||
|
|
||
| * The ``close()`` method (if exists) will be called if `open()` method exists and | ||
| returns successfully (irrespective of the return value), except if the Python | ||
| crashes in the middle. | ||
|
|
||
| .. note:: Evolving. | ||
|
|
||
| >>> # Print every row using a function | ||
| >>> def print_row(row): | ||
| ... print(row) | ||
| ... | ||
| >>> writer = sdf.writeStream.foreach(print_row) | ||
| >>> # Print every row using a object with process() method | ||
| >>> class RowPrinter: | ||
| ... def open(self, partition_id, epoch_id): | ||
| ... print("Opened %d, %d" % (partition_id, epoch_id)) | ||
| ... return True | ||
| ... def process(self, row): | ||
| ... print(row) | ||
| ... def close(self, error): | ||
| ... print("Closed with error: %s" % str(error)) | ||
| ... | ||
| >>> writer = sdf.writeStream.foreach(RowPrinter()) | ||
| """ | ||
|
|
||
| from pyspark.rdd import _wrap_function | ||
| from pyspark.serializers import PickleSerializer, AutoBatchedSerializer | ||
| from pyspark.taskcontext import TaskContext | ||
|
|
||
| if callable(f): | ||
| # The provided object is a callable function that is supposed to be called on each row. | ||
| # Construct a function that takes an iterator and calls the provided function on each | ||
| # row. | ||
| def func_without_process(_, iterator): | ||
| for x in iterator: | ||
| f(x) | ||
| return iter([]) | ||
|
|
||
| func = func_without_process | ||
|
|
||
| else: | ||
| # The provided object is not a callable function. Then it is expected to have a | ||
| # 'process(row)' method, and optional 'open(partition_id, epoch_id)' and | ||
| # 'close(error)' methods. | ||
|
|
||
| if not hasattr(f, 'process'): | ||
| raise Exception("Provided object does not have a 'process' method") | ||
|
|
||
| if not callable(getattr(f, 'process')): | ||
| raise Exception("Attribute 'process' in provided object is not callable") | ||
|
|
||
| def doesMethodExist(method_name): | ||
| exists = hasattr(f, method_name) | ||
| if exists and not callable(getattr(f, method_name)): | ||
| raise Exception( | ||
| "Attribute '%s' in provided object is not callable" % method_name) | ||
| return exists | ||
|
|
||
| open_exists = doesMethodExist('open') | ||
| close_exists = doesMethodExist('close') | ||
|
|
||
| def func_with_open_process_close(partition_id, iterator): | ||
| epoch_id = TaskContext.get().getLocalProperty('streaming.sql.batchId') | ||
| if epoch_id: | ||
| epoch_id = int(epoch_id) | ||
| else: | ||
| raise Exception("Could not get batch id from TaskContext") | ||
|
|
||
| # Check if the data should be processed | ||
| should_process = True | ||
| if open_exists: | ||
| should_process = f.open(partition_id, epoch_id) | ||
|
|
||
| error = None | ||
|
|
||
| try: | ||
| if should_process: | ||
| for x in iterator: | ||
| f.process(x) | ||
| except Exception as ex: | ||
| error = ex | ||
| finally: | ||
| if close_exists: | ||
| f.close(error) | ||
| if error: | ||
| raise error | ||
|
|
||
| return iter([]) | ||
|
|
||
| func = func_with_open_process_close | ||
|
|
||
| serializer = AutoBatchedSerializer(PickleSerializer()) | ||
| wrapped_func = _wrap_function(self._spark._sc, func, serializer, serializer) | ||
| jForeachWriter = \ | ||
| self._spark._sc._jvm.org.apache.spark.sql.execution.python.PythonForeachWriter( | ||
| wrapped_func, self._df._jdf.schema()) | ||
| self._jwrite.foreach(jForeachWriter) | ||
| return self | ||
|
|
||
| @ignore_unicode_prefix | ||
| @since(2.0) | ||
| def start(self, path=None, format=None, outputMode=None, partitionBy=None, queryName=None, | ||
|
|
||
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Seems this variant is specific to Python. I thought we should better match how we support with Scala side.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
This is superset of what we support in scala. Python users are more likely to use simple lambdas instead of defining classes. But they may also want to write transactional stuff in python with open and close methods. Hence providing both alternatives seems to be a good idea.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
(including the response to #21477 (comment)) I kind of agree that it's a-okay idea but I think we usually provide a consistent API support so far unless it's language specific, for example, ContextManager, decorator in Python and etc.
Just for clarification, does Scala side support function only support too?
Also, I know attribute-checking way is kind of more like "Pythonic" way but I am seeing the documentation is already diverted between Scala vs Python. It costs maintaining overhead on the other hand.
Uh oh!
There was an error while loading. Please reload this page.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I mean, we could maybe consider the other ways but wouldn't it better to have the consistent support as the primary, and then see if the other ways are really requested by users? I think we could still incrementally add attribute-checking way or the lambda (or function to be more correct) way later (but we can't in the opposite way).
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
Python APIs anyways have slightly divergences from Scala/Java APIs in order to provide better experiences for Python users. For example,
StreamingQuery.lastProgressreturns an object of typeStreamingQueryProgressin Java/Scala but returns a dict in python. Because python users are more used to dealing with dicts, and java/scala users (typed language) are more comfortable with structures). Similarly, in DataFrame.select, you can refer to columns in scala using$"columnName"but in python, you can refer to it asdf.columnName. If we strictly adhere to pure consistency, then we cannot make it convenient for users in different languages. And ultimately convenience is what matters for the user experience. So its okay to have a superset of supported types in python compared to java/scala.Personally, I think we should also add the lambda variant to Scala as well. But that decision for Scala is independent of this PR as there is enough justification for add the lambda variant for
Uh oh!
There was an error while loading. Please reload this page.
There was a problem hiding this comment.
Choose a reason for hiding this comment
The reason will be displayed to describe this comment to others. Learn more.
I believe
$"columnName"is more like a language specific feature in Scala and I thinkdf.columnNameis language specific to Python.Thing is, it sounded to me like we are kind of prejudging it.. We can't revert it back easily once we go in this way ..
+1
I am okay but I hope this shouldn't be usually done next time ..