-
Notifications
You must be signed in to change notification settings - Fork 29k
[SPARK-23446][PYTHON] Explicitly check supported types in toPandas #20625
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
Closed
HyukjinKwon
wants to merge
2
commits into
apache:master
from
HyukjinKwon:pandas_convertion_supported_type
Closed
Changes from 1 commit
Commits
Show all changes
2 commits
Select commit
Hold shift + click to select a range
File filter
Filter by extension
Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Next
Next commit
Explicitly specify supported types in toPandas
- Loading branch information
commit c79c6df7284b9717fe4e4c26090dcb51bf7712da
There are no files selected for viewing
This file contains hidden or bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
| Original file line number | Diff line number | Diff line change |
|---|---|---|
|
|
@@ -1988,10 +1988,11 @@ def toPandas(self): | |
| if self.sql_ctx.getConf("spark.sql.execution.arrow.enabled", "false").lower() == "true": | ||
| try: | ||
| from pyspark.sql.types import _check_dataframe_convert_date, \ | ||
| _check_dataframe_localize_timestamps | ||
| _check_dataframe_localize_timestamps, to_arrow_schema | ||
| from pyspark.sql.utils import require_minimum_pyarrow_version | ||
| import pyarrow | ||
| require_minimum_pyarrow_version() | ||
| to_arrow_schema(self.schema) | ||
| tables = self._collectAsArrow() | ||
| if tables: | ||
| table = pyarrow.concat_tables(tables) | ||
|
|
@@ -2000,10 +2001,12 @@ def toPandas(self): | |
| return _check_dataframe_localize_timestamps(pdf, timezone) | ||
| else: | ||
| return pd.DataFrame.from_records([], columns=self.columns) | ||
| except ImportError as e: | ||
| msg = "note: pyarrow must be installed and available on calling Python process " \ | ||
| "if using spark.sql.execution.arrow.enabled=true" | ||
| raise ImportError("%s\n%s" % (_exception_message(e), msg)) | ||
| except Exception as e: | ||
| msg = ( | ||
| "Note: toPandas attempted Arrow optimization because " | ||
| "'spark.sql.execution.arrow.enabled' is set to true. Please set it to false " | ||
| "to disable this.") | ||
| raise RuntimeError("%s\n%s" % (_exception_message(e), msg)) | ||
|
Member
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. Should the same type of error be raised instead of
Member
Author
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. Yup, please open a PR if you have a better idea. |
||
| else: | ||
| pdf = pd.DataFrame.from_records(self.collect(), columns=self.columns) | ||
|
|
||
|
|
||
This file contains hidden or bidirectional Unicode text that may be interpreted or compiled differently than what appears below. To review, open the file in an editor that reveals hidden Unicode characters.
Learn more about bidirectional Unicode characters
Add this suggestion to a batch that can be applied as a single commit.
This suggestion is invalid because no changes were made to the code.
Suggestions cannot be applied while the pull request is closed.
Suggestions cannot be applied while viewing a subset of changes.
Only one suggestion per line can be applied in a batch.
Add this suggestion to a batch that can be applied as a single commit.
Applying suggestions on deleted lines is not supported.
You must change the existing code in this line in order to create a valid suggestion.
Outdated suggestions cannot be applied.
This suggestion has been applied or marked resolved.
Suggestions cannot be applied from pending reviews.
Suggestions cannot be applied on multi-line comments.
Suggestions cannot be applied while the pull request is queued to merge.
Suggestion cannot be applied right now. Please check back later.
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.
hmm, this says why it's trying arrow and how to turn it off, but doesn't say why I have to turn it off? perhaps say something like pyarrow is not found (if it is the cause if we know)?
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.
Oh, that should be part of the original message. For example, I don't have PyArrow in
pypyin my local. it shows the error like: