| |
| <!DOCTYPE html> |
| |
| <html> |
| <head> |
| <meta charset="utf-8" /> |
| <title>Apache Arrow in PySpark — PySpark 3.3.4 documentation</title> |
| |
| <link rel="stylesheet" href="../../_static/css/index.73d71520a4ca3b99cfee5594769eaaae.css"> |
| |
| |
| <link rel="stylesheet" |
| href="../../_static/vendor/fontawesome/5.13.0/css/all.min.css"> |
| <link rel="preload" as="font" type="font/woff2" crossorigin |
| href="../../_static/vendor/fontawesome/5.13.0/webfonts/fa-solid-900.woff2"> |
| <link rel="preload" as="font" type="font/woff2" crossorigin |
| href="../../_static/vendor/fontawesome/5.13.0/webfonts/fa-brands-400.woff2"> |
| |
| |
| |
| <link rel="stylesheet" |
| href="../../_static/vendor/open-sans_all/1.44.1/index.css"> |
| <link rel="stylesheet" |
| href="../../_static/vendor/lato_latin-ext/1.44.1/index.css"> |
| |
| |
| <link rel="stylesheet" href="../../_static/basic.css" type="text/css" /> |
| <link rel="stylesheet" href="../../_static/pygments.css" type="text/css" /> |
| <link rel="stylesheet" type="text/css" href="../../_static/copybutton.css" /> |
| <link rel="stylesheet" type="text/css" href="../../_static/css/pyspark.css" /> |
| |
| <link rel="preload" as="script" href="../../_static/js/index.3da636dd464baa7582d2.js"> |
| |
| <script id="documentation_options" data-url_root="../../" src="../../_static/documentation_options.js"></script> |
| <script src="../../_static/jquery.js"></script> |
| <script src="../../_static/underscore.js"></script> |
| <script src="../../_static/doctools.js"></script> |
| <script src="../../_static/language_data.js"></script> |
| <script src="../../_static/clipboard.min.js"></script> |
| <script src="../../_static/copybutton.js"></script> |
| <script crossorigin="anonymous" integrity="sha256-Ae2Vz/4ePdIu6ZyI/5ZGsYnb+m0JlOmKPjt6XZ9JJkA=" src="https://cdnjs.cloudflare.com/ajax/libs/require.js/2.3.4/require.min.js"></script> |
| <script async="async" src="https://cdnjs.cloudflare.com/ajax/libs/mathjax/2.7.5/latest.js?config=TeX-AMS-MML_HTMLorMML"></script> |
| <script type="text/x-mathjax-config">MathJax.Hub.Config({"tex2jax": {"inlineMath": [["$", "$"], ["\\(", "\\)"]], "processEscapes": true, "ignoreClass": "document", "processClass": "math|output_area"}})</script> |
| <link rel="canonical" href="https://spark.apache.org/docs/latest/api/python/user_guide/sql/arrow_pandas.html" /> |
| <link rel="search" title="Search" href="../../search.html" /> |
| <link rel="next" title="Pandas API on Spark" href="../pandas_on_spark/index.html" /> |
| <link rel="prev" title="Spark SQL" href="index.html" /> |
| <meta name="viewport" content="width=device-width, initial-scale=1" /> |
| <meta name="docsearch:language" content="en" /> |
| </head> |
| <body data-spy="scroll" data-target="#bd-toc-nav" data-offset="80"> |
| |
| <nav class="navbar navbar-light navbar-expand-lg bg-light fixed-top bd-navbar" id="navbar-main"> |
| <div class="container-xl"> |
| |
| <a class="navbar-brand" href="../../index.html"> |
| |
| <img src="../../_static/spark-logo-reverse.png" class="logo" alt="logo" /> |
| |
| </a> |
| <button class="navbar-toggler" type="button" data-toggle="collapse" data-target="#navbar-menu" aria-controls="navbar-menu" aria-expanded="false" aria-label="Toggle navigation"> |
| <span class="navbar-toggler-icon"></span> |
| </button> |
| |
| <div id="navbar-menu" class="col-lg-9 collapse navbar-collapse"> |
| <ul id="navbar-main-elements" class="navbar-nav mr-auto"> |
| |
| |
| <li class="nav-item "> |
| <a class="nav-link" href="../../getting_started/index.html">Getting Started</a> |
| </li> |
| |
| <li class="nav-item active"> |
| <a class="nav-link" href="../index.html">User Guide</a> |
| </li> |
| |
| <li class="nav-item "> |
| <a class="nav-link" href="../../reference/index.html">API Reference</a> |
| </li> |
| |
| <li class="nav-item "> |
| <a class="nav-link" href="../../development/index.html">Development</a> |
| </li> |
| |
| <li class="nav-item "> |
| <a class="nav-link" href="../../migration_guide/index.html">Migration Guide</a> |
| </li> |
| |
| |
| </ul> |
| |
| |
| |
| |
| <ul class="navbar-nav"> |
| |
| |
| </ul> |
| </div> |
| </div> |
| </nav> |
| |
| |
| <div class="container-xl"> |
| <div class="row"> |
| |
| <div class="col-12 col-md-3 bd-sidebar"><form class="bd-search d-flex align-items-center" action="../../search.html" method="get"> |
| <i class="icon fas fa-search"></i> |
| <input type="search" class="form-control" name="q" id="search-input" placeholder="Search the docs ..." aria-label="Search the docs ..." autocomplete="off" > |
| </form> |
| <nav class="bd-links" id="bd-docs-nav" aria-label="Main navigation"> |
| |
| <div class="bd-toc-item active"> |
| |
| |
| <ul class="nav bd-sidenav"> |
| |
| |
| |
| |
| |
| |
| <li class=""> |
| <a href="../python_packaging.html">Python Package Management</a> |
| </li> |
| |
| |
| |
| |
| <li class="active"> |
| <a href="index.html">Spark SQL</a> |
| <ul> |
| |
| <li class="active"> |
| <a href="">Apache Arrow in PySpark</a> |
| </li> |
| |
| </ul> |
| </li> |
| |
| |
| |
| <li class=""> |
| <a href="../pandas_on_spark/index.html">Pandas API on Spark</a> |
| </li> |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| |
| </ul> |
| |
| </nav> |
| </div> |
| |
| |
| |
| <div class="d-none d-xl-block col-xl-2 bd-toc"> |
| |
| <div class="tocsection onthispage pt-5 pb-3"> |
| <i class="fas fa-list"></i> On this page |
| </div> |
| |
| <nav id="bd-toc-nav"> |
| <ul class="nav section-nav flex-column"> |
| |
| <li class="nav-item toc-entry toc-h2"> |
| <a href="#ensure-pyarrow-installed" class="nav-link">Ensure PyArrow Installed</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h2"> |
| <a href="#enabling-for-conversion-to-from-pandas" class="nav-link">Enabling for Conversion to/from Pandas</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h2"> |
| <a href="#pandas-udfs-a-k-a-vectorized-udfs" class="nav-link">Pandas UDFs (a.k.a. Vectorized UDFs)</a><ul class="nav section-nav flex-column"> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#series-to-series" class="nav-link">Series to Series</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#iterator-of-series-to-iterator-of-series" class="nav-link">Iterator of Series to Iterator of Series</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#iterator-of-multiple-series-to-iterator-of-series" class="nav-link">Iterator of Multiple Series to Iterator of Series</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#series-to-scalar" class="nav-link">Series to Scalar</a> |
| </li> |
| |
| </ul> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h2"> |
| <a href="#pandas-function-apis" class="nav-link">Pandas Function APIs</a><ul class="nav section-nav flex-column"> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#grouped-map" class="nav-link">Grouped Map</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#map" class="nav-link">Map</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#co-grouped-map" class="nav-link">Co-grouped Map</a> |
| </li> |
| |
| </ul> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h2"> |
| <a href="#usage-notes" class="nav-link">Usage Notes</a><ul class="nav section-nav flex-column"> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#supported-sql-types" class="nav-link">Supported SQL Types</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#setting-arrow-batch-size" class="nav-link">Setting Arrow Batch Size</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#timestamp-with-time-zone-semantics" class="nav-link">Timestamp with Time Zone Semantics</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#recommended-pandas-and-pyarrow-versions" class="nav-link">Recommended Pandas and PyArrow Versions</a> |
| </li> |
| |
| <li class="nav-item toc-entry toc-h3"> |
| <a href="#setting-arrow-self-destruct-for-memory-savings" class="nav-link">Setting Arrow self_destruct for memory savings</a> |
| </li> |
| |
| </ul> |
| </li> |
| |
| </ul> |
| </nav> |
| |
| |
| |
| </div> |
| |
| |
| |
| <main class="col-12 col-md-9 col-xl-7 py-md-5 pl-md-5 pr-md-4 bd-content" role="main"> |
| |
| <div> |
| |
| <div class="section" id="apache-arrow-in-pyspark"> |
| <h1>Apache Arrow in PySpark<a class="headerlink" href="#apache-arrow-in-pyspark" title="Permalink to this headline">¶</a></h1> |
| <p>Apache Arrow is an in-memory columnar data format that is used in Spark to efficiently transfer |
| data between JVM and Python processes. This currently is most beneficial to Python users that |
| work with Pandas/NumPy data. Its usage is not automatic and might require some minor |
| changes to configuration or code to take full advantage and ensure compatibility. This guide will |
| give a high-level description of how to use Arrow in Spark and highlight any differences when |
| working with Arrow-enabled data.</p> |
| <div class="section" id="ensure-pyarrow-installed"> |
| <h2>Ensure PyArrow Installed<a class="headerlink" href="#ensure-pyarrow-installed" title="Permalink to this headline">¶</a></h2> |
| <p>To use Apache Arrow in PySpark, <a class="reference internal" href="#recommended-pandas-and-pyarrow-versions"><span class="std std-ref">the recommended version of PyArrow</span></a> |
| should be installed. |
| If you install PySpark using pip, then PyArrow can be brought in as an extra dependency of the |
| SQL module with the command <code class="docutils literal notranslate"><span class="pre">pip</span> <span class="pre">install</span> <span class="pre">pyspark[sql]</span></code>. Otherwise, you must ensure that PyArrow |
| is installed and available on all cluster nodes. |
| You can install using pip or conda from the conda-forge channel. See PyArrow |
| <a class="reference external" href="https://arrow.apache.org/docs/python/install.html">installation</a> for details.</p> |
| </div> |
| <div class="section" id="enabling-for-conversion-to-from-pandas"> |
| <h2>Enabling for Conversion to/from Pandas<a class="headerlink" href="#enabling-for-conversion-to-from-pandas" title="Permalink to this headline">¶</a></h2> |
| <p>Arrow is available as an optimization when converting a Spark DataFrame to a Pandas DataFrame |
| using the call <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.toPandas.html#pyspark.sql.DataFrame.toPandas" title="pyspark.sql.DataFrame.toPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">DataFrame.toPandas()</span></code></a> and when creating a Spark DataFrame from a Pandas DataFrame with |
| <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.SparkSession.createDataFrame.html#pyspark.sql.SparkSession.createDataFrame" title="pyspark.sql.SparkSession.createDataFrame"><code class="xref py py-meth docutils literal notranslate"><span class="pre">SparkSession.createDataFrame()</span></code></a>. To use Arrow when executing these calls, users need to first set |
| the Spark configuration <code class="docutils literal notranslate"><span class="pre">spark.sql.execution.arrow.pyspark.enabled</span></code> to <code class="docutils literal notranslate"><span class="pre">true</span></code>. This is disabled by default.</p> |
| <p>In addition, optimizations enabled by <code class="docutils literal notranslate"><span class="pre">spark.sql.execution.arrow.pyspark.enabled</span></code> could fallback automatically |
| to non-Arrow optimization implementation if an error occurs before the actual computation within Spark. |
| This can be controlled by <code class="docutils literal notranslate"><span class="pre">spark.sql.execution.arrow.pyspark.fallback.enabled</span></code>.</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="kn">import</span> <span class="nn">numpy</span> <span class="k">as</span> <span class="nn">np</span> |
| <span class="kn">import</span> <span class="nn">pandas</span> <span class="k">as</span> <span class="nn">pd</span> |
| |
| <span class="c1"># Enable Arrow-based columnar data transfers</span> |
| <span class="n">spark</span><span class="o">.</span><span class="n">conf</span><span class="o">.</span><span class="n">set</span><span class="p">(</span><span class="s2">"spark.sql.execution.arrow.pyspark.enabled"</span><span class="p">,</span> <span class="s2">"true"</span><span class="p">)</span> |
| |
| <span class="c1"># Generate a Pandas DataFrame</span> |
| <span class="n">pdf</span> <span class="o">=</span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">(</span><span class="n">np</span><span class="o">.</span><span class="n">random</span><span class="o">.</span><span class="n">rand</span><span class="p">(</span><span class="mi">100</span><span class="p">,</span> <span class="mi">3</span><span class="p">))</span> |
| |
| <span class="c1"># Create a Spark DataFrame from a Pandas DataFrame using Arrow</span> |
| <span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span><span class="n">pdf</span><span class="p">)</span> |
| |
| <span class="c1"># Convert the Spark DataFrame back to a Pandas DataFrame using Arrow</span> |
| <span class="n">result_pdf</span> <span class="o">=</span> <span class="n">df</span><span class="o">.</span><span class="n">select</span><span class="p">(</span><span class="s2">"*"</span><span class="p">)</span><span class="o">.</span><span class="n">toPandas</span><span class="p">()</span> |
| |
| <span class="nb">print</span><span class="p">(</span><span class="s2">"Pandas DataFrame result statistics:</span><span class="se">\n</span><span class="si">%s</span><span class="se">\n</span><span class="s2">"</span> <span class="o">%</span> <span class="nb">str</span><span class="p">(</span><span class="n">result_pdf</span><span class="o">.</span><span class="n">describe</span><span class="p">()))</span> |
| </pre></div> |
| </div> |
| <p>Using the above optimizations with Arrow will produce the same results as when Arrow is not |
| enabled.</p> |
| <p>Note that even with Arrow, <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.toPandas.html#pyspark.sql.DataFrame.toPandas" title="pyspark.sql.DataFrame.toPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">DataFrame.toPandas()</span></code></a> results in the collection of all records in the |
| DataFrame to the driver program and should be done on a small subset of the data. Not all Spark |
| data types are currently supported and an error can be raised if a column has an unsupported type. |
| If an error occurs during <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.SparkSession.createDataFrame.html#pyspark.sql.SparkSession.createDataFrame" title="pyspark.sql.SparkSession.createDataFrame"><code class="xref py py-meth docutils literal notranslate"><span class="pre">SparkSession.createDataFrame()</span></code></a>, Spark will fall back to create the |
| DataFrame without Arrow.</p> |
| </div> |
| <div class="section" id="pandas-udfs-a-k-a-vectorized-udfs"> |
| <h2>Pandas UDFs (a.k.a. Vectorized UDFs)<a class="headerlink" href="#pandas-udfs-a-k-a-vectorized-udfs" title="Permalink to this headline">¶</a></h2> |
| <p>Pandas UDFs are user defined functions that are executed by Spark using |
| Arrow to transfer data and Pandas to work with the data, which allows vectorized operations. A Pandas |
| UDF is defined using the <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-meth docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a> as a decorator or to wrap the function, and no additional |
| configuration is required. A Pandas UDF behaves as a regular PySpark function API in general.</p> |
| <p>Before Spark 3.0, Pandas UDFs used to be defined with <code class="docutils literal notranslate"><span class="pre">pyspark.sql.functions.PandasUDFType</span></code>. From Spark 3.0 |
| with Python 3.6+, you can also use <a class="reference external" href="https://www.python.org/dev/peps/pep-0484">Python type hints</a>. |
| Using Python type hints is preferred and using <code class="docutils literal notranslate"><span class="pre">pyspark.sql.functions.PandasUDFType</span></code> will be deprecated in |
| the future release.</p> |
| <p>Note that the type hint should use <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code> in all cases but there is one variant |
| that <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> should be used for its input or output type hint instead when the input |
| or output column is of <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.types.StructType.html#pyspark.sql.types.StructType" title="pyspark.sql.types.StructType"><code class="xref py py-class docutils literal notranslate"><span class="pre">StructType</span></code></a>. The following example shows a Pandas UDF which takes long |
| column, string column and struct column, and outputs a struct column. It requires the function to |
| specify the type hints of <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code> and <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> as below:</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="kn">import</span> <span class="nn">pandas</span> <span class="k">as</span> <span class="nn">pd</span> |
| |
| <span class="kn">from</span> <span class="nn">pyspark.sql.functions</span> <span class="kn">import</span> <span class="n">pandas_udf</span> |
| |
| <span class="nd">@pandas_udf</span><span class="p">(</span><span class="s2">"col1 string, col2 long"</span><span class="p">)</span> <span class="c1"># type: ignore[call-overload]</span> |
| <span class="k">def</span> <span class="nf">func</span><span class="p">(</span><span class="n">s1</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">,</span> <span class="n">s2</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">,</span> <span class="n">s3</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">)</span> <span class="o">-></span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">:</span> |
| <span class="n">s3</span><span class="p">[</span><span class="s1">'col2'</span><span class="p">]</span> <span class="o">=</span> <span class="n">s1</span> <span class="o">+</span> <span class="n">s2</span><span class="o">.</span><span class="n">str</span><span class="o">.</span><span class="n">len</span><span class="p">()</span> |
| <span class="k">return</span> <span class="n">s3</span> |
| |
| <span class="c1"># Create a Spark DataFrame that has three columns including a struct column.</span> |
| <span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span> |
| <span class="p">[[</span><span class="mi">1</span><span class="p">,</span> <span class="s2">"a string"</span><span class="p">,</span> <span class="p">(</span><span class="s2">"a nested string"</span><span class="p">,)]],</span> |
| <span class="s2">"long_col long, string_col string, struct_col struct<col1:string>"</span><span class="p">)</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">printSchema</span><span class="p">()</span> |
| <span class="c1"># root</span> |
| <span class="c1"># |-- long_column: long (nullable = true)</span> |
| <span class="c1"># |-- string_column: string (nullable = true)</span> |
| <span class="c1"># |-- struct_column: struct (nullable = true)</span> |
| <span class="c1"># | |-- col1: string (nullable = true)</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">select</span><span class="p">(</span><span class="n">func</span><span class="p">(</span><span class="s2">"long_col"</span><span class="p">,</span> <span class="s2">"string_col"</span><span class="p">,</span> <span class="s2">"struct_col"</span><span class="p">))</span><span class="o">.</span><span class="n">printSchema</span><span class="p">()</span> |
| <span class="c1"># |-- func(long_col, string_col, struct_col): struct (nullable = true)</span> |
| <span class="c1"># | |-- col1: string (nullable = true)</span> |
| <span class="c1"># | |-- col2: long (nullable = true)</span> |
| </pre></div> |
| </div> |
| <p>In the following sections, it describes the combinations of the supported type hints. For simplicity, |
| <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> variant is omitted.</p> |
| <div class="section" id="series-to-series"> |
| <h3>Series to Series<a class="headerlink" href="#series-to-series" title="Permalink to this headline">¶</a></h3> |
| <p>The type hint can be expressed as <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code>, … -> <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code>.</p> |
| <p>By using <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a> with the function having such type hints above, it creates a Pandas UDF where the given |
| function takes one or more <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code> and outputs one <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code>. The output of the function should |
| always be of the same length as the input. Internally, PySpark will execute a Pandas UDF by splitting |
| columns into batches and calling the function for each batch as a subset of the data, then concatenating |
| the results together.</p> |
| <p>The following example shows how to create this Pandas UDF that computes the product of 2 columns.</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="kn">import</span> <span class="nn">pandas</span> <span class="k">as</span> <span class="nn">pd</span> |
| |
| <span class="kn">from</span> <span class="nn">pyspark.sql.functions</span> <span class="kn">import</span> <span class="n">col</span><span class="p">,</span> <span class="n">pandas_udf</span> |
| <span class="kn">from</span> <span class="nn">pyspark.sql.types</span> <span class="kn">import</span> <span class="n">LongType</span> |
| |
| <span class="c1"># Declare the function and create the UDF</span> |
| <span class="k">def</span> <span class="nf">multiply_func</span><span class="p">(</span><span class="n">a</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">,</span> <span class="n">b</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">)</span> <span class="o">-></span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">a</span> <span class="o">*</span> <span class="n">b</span> |
| |
| <span class="n">multiply</span> <span class="o">=</span> <span class="n">pandas_udf</span><span class="p">(</span><span class="n">multiply_func</span><span class="p">,</span> <span class="n">returnType</span><span class="o">=</span><span class="n">LongType</span><span class="p">())</span> <span class="c1"># type: ignore[call-overload]</span> |
| |
| <span class="c1"># The function for a pandas_udf should be able to execute with local Pandas data</span> |
| <span class="n">x</span> <span class="o">=</span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">([</span><span class="mi">1</span><span class="p">,</span> <span class="mi">2</span><span class="p">,</span> <span class="mi">3</span><span class="p">])</span> |
| <span class="nb">print</span><span class="p">(</span><span class="n">multiply_func</span><span class="p">(</span><span class="n">x</span><span class="p">,</span> <span class="n">x</span><span class="p">))</span> |
| <span class="c1"># 0 1</span> |
| <span class="c1"># 1 4</span> |
| <span class="c1"># 2 9</span> |
| <span class="c1"># dtype: int64</span> |
| |
| <span class="c1"># Create a Spark DataFrame, 'spark' is an existing SparkSession</span> |
| <span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span><span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">(</span><span class="n">x</span><span class="p">,</span> <span class="n">columns</span><span class="o">=</span><span class="p">[</span><span class="s2">"x"</span><span class="p">]))</span> |
| |
| <span class="c1"># Execute function as a Spark vectorized UDF</span> |
| <span class="n">df</span><span class="o">.</span><span class="n">select</span><span class="p">(</span><span class="n">multiply</span><span class="p">(</span><span class="n">col</span><span class="p">(</span><span class="s2">"x"</span><span class="p">),</span> <span class="n">col</span><span class="p">(</span><span class="s2">"x"</span><span class="p">)))</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +-------------------+</span> |
| <span class="c1"># |multiply_func(x, x)|</span> |
| <span class="c1"># +-------------------+</span> |
| <span class="c1"># | 1|</span> |
| <span class="c1"># | 4|</span> |
| <span class="c1"># | 9|</span> |
| <span class="c1"># +-------------------+</span> |
| </pre></div> |
| </div> |
| <p>For detailed usage, please see <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a>.</p> |
| </div> |
| <div class="section" id="iterator-of-series-to-iterator-of-series"> |
| <h3>Iterator of Series to Iterator of Series<a class="headerlink" href="#iterator-of-series-to-iterator-of-series" title="Permalink to this headline">¶</a></h3> |
| <p>The type hint can be expressed as <code class="docutils literal notranslate"><span class="pre">Iterator[pandas.Series]</span></code> -> <code class="docutils literal notranslate"><span class="pre">Iterator[pandas.Series]</span></code>.</p> |
| <p>By using <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a> with the function having such type hints above, it creates a Pandas UDF where the given |
| function takes an iterator of <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code> and outputs an iterator of <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code>. The |
| length of the entire output from the function should be the same length of the entire input; therefore, it can |
| prefetch the data from the input iterator as long as the lengths are the same. |
| In this case, the created Pandas UDF requires one input column when the Pandas UDF is called. To use |
| multiple input columns, a different type hint is required. See Iterator of Multiple Series to Iterator |
| of Series.</p> |
| <p>It is also useful when the UDF execution requires initializing some states although internally it works |
| identically as Series to Series case. The pseudocode below illustrates the example.</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="nd">@pandas_udf</span><span class="p">(</span><span class="s2">"long"</span><span class="p">)</span> |
| <span class="k">def</span> <span class="nf">calculate</span><span class="p">(</span><span class="n">iterator</span><span class="p">:</span> <span class="n">Iterator</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">])</span> <span class="o">-></span> <span class="n">Iterator</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">]:</span> |
| <span class="c1"># Do some expensive initialization with a state</span> |
| <span class="n">state</span> <span class="o">=</span> <span class="n">very_expensive_initialization</span><span class="p">()</span> |
| <span class="k">for</span> <span class="n">x</span> <span class="ow">in</span> <span class="n">iterator</span><span class="p">:</span> |
| <span class="c1"># Use that state for whole iterator.</span> |
| <span class="k">yield</span> <span class="n">calculate_with_state</span><span class="p">(</span><span class="n">x</span><span class="p">,</span> <span class="n">state</span><span class="p">)</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">select</span><span class="p">(</span><span class="n">calculate</span><span class="p">(</span><span class="s2">"value"</span><span class="p">))</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| </pre></div> |
| </div> |
| <p>The following example shows how to create this Pandas UDF:</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Iterator</span> |
| |
| <span class="kn">import</span> <span class="nn">pandas</span> <span class="k">as</span> <span class="nn">pd</span> |
| |
| <span class="kn">from</span> <span class="nn">pyspark.sql.functions</span> <span class="kn">import</span> <span class="n">pandas_udf</span> |
| |
| <span class="n">pdf</span> <span class="o">=</span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">([</span><span class="mi">1</span><span class="p">,</span> <span class="mi">2</span><span class="p">,</span> <span class="mi">3</span><span class="p">],</span> <span class="n">columns</span><span class="o">=</span><span class="p">[</span><span class="s2">"x"</span><span class="p">])</span> |
| <span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span><span class="n">pdf</span><span class="p">)</span> |
| |
| <span class="c1"># Declare the function and create the UDF</span> |
| <span class="nd">@pandas_udf</span><span class="p">(</span><span class="s2">"long"</span><span class="p">)</span> <span class="c1"># type: ignore[call-overload]</span> |
| <span class="k">def</span> <span class="nf">plus_one</span><span class="p">(</span><span class="n">iterator</span><span class="p">:</span> <span class="n">Iterator</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">])</span> <span class="o">-></span> <span class="n">Iterator</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">]:</span> |
| <span class="k">for</span> <span class="n">x</span> <span class="ow">in</span> <span class="n">iterator</span><span class="p">:</span> |
| <span class="k">yield</span> <span class="n">x</span> <span class="o">+</span> <span class="mi">1</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">select</span><span class="p">(</span><span class="n">plus_one</span><span class="p">(</span><span class="s2">"x"</span><span class="p">))</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +-----------+</span> |
| <span class="c1"># |plus_one(x)|</span> |
| <span class="c1"># +-----------+</span> |
| <span class="c1"># | 2|</span> |
| <span class="c1"># | 3|</span> |
| <span class="c1"># | 4|</span> |
| <span class="c1"># +-----------+</span> |
| </pre></div> |
| </div> |
| <p>For detailed usage, please see <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a>.</p> |
| </div> |
| <div class="section" id="iterator-of-multiple-series-to-iterator-of-series"> |
| <h3>Iterator of Multiple Series to Iterator of Series<a class="headerlink" href="#iterator-of-multiple-series-to-iterator-of-series" title="Permalink to this headline">¶</a></h3> |
| <p>The type hint can be expressed as <code class="docutils literal notranslate"><span class="pre">Iterator[Tuple[pandas.Series,</span> <span class="pre">...]]</span></code> -> <code class="docutils literal notranslate"><span class="pre">Iterator[pandas.Series]</span></code>.</p> |
| <p>By using <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a> with the function having such type hints above, it creates a Pandas UDF where the |
| given function takes an iterator of a tuple of multiple <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code> and outputs an iterator of <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code>. |
| In this case, the created pandas UDF requires multiple input columns as many as the series in the tuple |
| when the Pandas UDF is called. Otherwise, it has the same characteristics and restrictions as Iterator of Series |
| to Iterator of Series case.</p> |
| <p>The following example shows how to create this Pandas UDF:</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="kn">from</span> <span class="nn">typing</span> <span class="kn">import</span> <span class="n">Iterator</span><span class="p">,</span> <span class="n">Tuple</span> |
| |
| <span class="kn">import</span> <span class="nn">pandas</span> <span class="k">as</span> <span class="nn">pd</span> |
| |
| <span class="kn">from</span> <span class="nn">pyspark.sql.functions</span> <span class="kn">import</span> <span class="n">pandas_udf</span> |
| |
| <span class="n">pdf</span> <span class="o">=</span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">([</span><span class="mi">1</span><span class="p">,</span> <span class="mi">2</span><span class="p">,</span> <span class="mi">3</span><span class="p">],</span> <span class="n">columns</span><span class="o">=</span><span class="p">[</span><span class="s2">"x"</span><span class="p">])</span> |
| <span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span><span class="n">pdf</span><span class="p">)</span> |
| |
| <span class="c1"># Declare the function and create the UDF</span> |
| <span class="nd">@pandas_udf</span><span class="p">(</span><span class="s2">"long"</span><span class="p">)</span> <span class="c1"># type: ignore[call-overload]</span> |
| <span class="k">def</span> <span class="nf">multiply_two_cols</span><span class="p">(</span> |
| <span class="n">iterator</span><span class="p">:</span> <span class="n">Iterator</span><span class="p">[</span><span class="n">Tuple</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">,</span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">]])</span> <span class="o">-></span> <span class="n">Iterator</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">]:</span> |
| <span class="k">for</span> <span class="n">a</span><span class="p">,</span> <span class="n">b</span> <span class="ow">in</span> <span class="n">iterator</span><span class="p">:</span> |
| <span class="k">yield</span> <span class="n">a</span> <span class="o">*</span> <span class="n">b</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">select</span><span class="p">(</span><span class="n">multiply_two_cols</span><span class="p">(</span><span class="s2">"x"</span><span class="p">,</span> <span class="s2">"x"</span><span class="p">))</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +-----------------------+</span> |
| <span class="c1"># |multiply_two_cols(x, x)|</span> |
| <span class="c1"># +-----------------------+</span> |
| <span class="c1"># | 1|</span> |
| <span class="c1"># | 4|</span> |
| <span class="c1"># | 9|</span> |
| <span class="c1"># +-----------------------+</span> |
| </pre></div> |
| </div> |
| <p>For detailed usage, please see <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a>.</p> |
| </div> |
| <div class="section" id="series-to-scalar"> |
| <h3>Series to Scalar<a class="headerlink" href="#series-to-scalar" title="Permalink to this headline">¶</a></h3> |
| <p>The type hint can be expressed as <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code>, … -> <code class="docutils literal notranslate"><span class="pre">Any</span></code>.</p> |
| <p>By using <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a> with the function having such type hints above, it creates a Pandas UDF similar |
| to PySpark’s aggregate functions. The given function takes <cite>pandas.Series</cite> and returns a scalar value. |
| The return type should be a primitive data type, and the returned scalar can be either a python |
| primitive type, e.g., <code class="docutils literal notranslate"><span class="pre">int</span></code> or <code class="docutils literal notranslate"><span class="pre">float</span></code> or a numpy data type, e.g., <code class="docutils literal notranslate"><span class="pre">numpy.int64</span></code> or <code class="docutils literal notranslate"><span class="pre">numpy.float64</span></code>. |
| <code class="docutils literal notranslate"><span class="pre">Any</span></code> should ideally be a specific scalar type accordingly.</p> |
| <p>This UDF can be also used with <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.GroupedData.agg.html#pyspark.sql.GroupedData.agg" title="pyspark.sql.GroupedData.agg"><code class="xref py py-meth docutils literal notranslate"><span class="pre">GroupedData.agg()</span></code></a> and <cite>Window</cite>. |
| It defines an aggregation from one or more <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code> to a scalar value, where each <code class="docutils literal notranslate"><span class="pre">pandas.Series</span></code> |
| represents a column within the group or window.</p> |
| <p>Note that this type of UDF does not support partial aggregation and all data for a group or window |
| will be loaded into memory. Also, only unbounded window is supported with Grouped aggregate Pandas |
| UDFs currently. The following example shows how to use this type of UDF to compute mean with a group-by |
| and window operations:</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="kn">import</span> <span class="nn">pandas</span> <span class="k">as</span> <span class="nn">pd</span> |
| |
| <span class="kn">from</span> <span class="nn">pyspark.sql.functions</span> <span class="kn">import</span> <span class="n">pandas_udf</span> |
| <span class="kn">from</span> <span class="nn">pyspark.sql</span> <span class="kn">import</span> <span class="n">Window</span> |
| |
| <span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span> |
| <span class="p">[(</span><span class="mi">1</span><span class="p">,</span> <span class="mf">1.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">1</span><span class="p">,</span> <span class="mf">2.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">2</span><span class="p">,</span> <span class="mf">3.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">2</span><span class="p">,</span> <span class="mf">5.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">2</span><span class="p">,</span> <span class="mf">10.0</span><span class="p">)],</span> |
| <span class="p">(</span><span class="s2">"id"</span><span class="p">,</span> <span class="s2">"v"</span><span class="p">))</span> |
| |
| <span class="c1"># Declare the function and create the UDF</span> |
| <span class="nd">@pandas_udf</span><span class="p">(</span><span class="s2">"double"</span><span class="p">)</span> <span class="c1"># type: ignore[call-overload]</span> |
| <span class="k">def</span> <span class="nf">mean_udf</span><span class="p">(</span><span class="n">v</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">Series</span><span class="p">)</span> <span class="o">-></span> <span class="nb">float</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">v</span><span class="o">.</span><span class="n">mean</span><span class="p">()</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">select</span><span class="p">(</span><span class="n">mean_udf</span><span class="p">(</span><span class="n">df</span><span class="p">[</span><span class="s1">'v'</span><span class="p">]))</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +-----------+</span> |
| <span class="c1"># |mean_udf(v)|</span> |
| <span class="c1"># +-----------+</span> |
| <span class="c1"># | 4.2|</span> |
| <span class="c1"># +-----------+</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">groupby</span><span class="p">(</span><span class="s2">"id"</span><span class="p">)</span><span class="o">.</span><span class="n">agg</span><span class="p">(</span><span class="n">mean_udf</span><span class="p">(</span><span class="n">df</span><span class="p">[</span><span class="s1">'v'</span><span class="p">]))</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +---+-----------+</span> |
| <span class="c1"># | id|mean_udf(v)|</span> |
| <span class="c1"># +---+-----------+</span> |
| <span class="c1"># | 1| 1.5|</span> |
| <span class="c1"># | 2| 6.0|</span> |
| <span class="c1"># +---+-----------+</span> |
| |
| <span class="n">w</span> <span class="o">=</span> <span class="n">Window</span> \ |
| <span class="o">.</span><span class="n">partitionBy</span><span class="p">(</span><span class="s1">'id'</span><span class="p">)</span> \ |
| <span class="o">.</span><span class="n">rowsBetween</span><span class="p">(</span><span class="n">Window</span><span class="o">.</span><span class="n">unboundedPreceding</span><span class="p">,</span> <span class="n">Window</span><span class="o">.</span><span class="n">unboundedFollowing</span><span class="p">)</span> |
| <span class="n">df</span><span class="o">.</span><span class="n">withColumn</span><span class="p">(</span><span class="s1">'mean_v'</span><span class="p">,</span> <span class="n">mean_udf</span><span class="p">(</span><span class="n">df</span><span class="p">[</span><span class="s1">'v'</span><span class="p">])</span><span class="o">.</span><span class="n">over</span><span class="p">(</span><span class="n">w</span><span class="p">))</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +---+----+------+</span> |
| <span class="c1"># | id| v|mean_v|</span> |
| <span class="c1"># +---+----+------+</span> |
| <span class="c1"># | 1| 1.0| 1.5|</span> |
| <span class="c1"># | 1| 2.0| 1.5|</span> |
| <span class="c1"># | 2| 3.0| 6.0|</span> |
| <span class="c1"># | 2| 5.0| 6.0|</span> |
| <span class="c1"># | 2|10.0| 6.0|</span> |
| <span class="c1"># +---+----+------+</span> |
| </pre></div> |
| </div> |
| <p>For detailed usage, please see <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.functions.pandas_udf.html#pyspark.sql.functions.pandas_udf" title="pyspark.sql.functions.pandas_udf"><code class="xref py py-func docutils literal notranslate"><span class="pre">pandas_udf()</span></code></a>.</p> |
| </div> |
| </div> |
| <div class="section" id="pandas-function-apis"> |
| <h2>Pandas Function APIs<a class="headerlink" href="#pandas-function-apis" title="Permalink to this headline">¶</a></h2> |
| <p>Pandas Function APIs can directly apply a Python native function against the whole <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a> by |
| using Pandas instances. Internally it works similarly with Pandas UDFs by using Arrow to transfer |
| data and Pandas to work with the data, which allows vectorized operations. However, a Pandas Function |
| API behaves as a regular API under PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a> instead of <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.Column.html#pyspark.sql.Column" title="pyspark.sql.Column"><code class="xref py py-class docutils literal notranslate"><span class="pre">Column</span></code></a>, and Python type hints in Pandas |
| Functions APIs are optional and do not affect how it works internally at this moment although they |
| might be required in the future.</p> |
| <p>From Spark 3.0, grouped map pandas UDF is now categorized as a separate Pandas Function API, |
| <code class="docutils literal notranslate"><span class="pre">DataFrame.groupby().applyInPandas()</span></code>. It is still possible to use it with <code class="docutils literal notranslate"><span class="pre">pyspark.sql.functions.PandasUDFType</span></code> |
| and <code class="docutils literal notranslate"><span class="pre">DataFrame.groupby().apply()</span></code> as it was; however, it is preferred to use |
| <code class="docutils literal notranslate"><span class="pre">DataFrame.groupby().applyInPandas()</span></code> directly. Using <code class="docutils literal notranslate"><span class="pre">pyspark.sql.functions.PandasUDFType</span></code> will be deprecated |
| in the future.</p> |
| <div class="section" id="grouped-map"> |
| <h3>Grouped Map<a class="headerlink" href="#grouped-map" title="Permalink to this headline">¶</a></h3> |
| <p>Grouped map operations with Pandas instances are supported by <code class="docutils literal notranslate"><span class="pre">DataFrame.groupby().applyInPandas()</span></code> |
| which requires a Python function that takes a <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> and return another <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>. |
| It maps each group to each <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> in the Python function.</p> |
| <p>This API implements the “split-apply-combine” pattern which consists of three steps:</p> |
| <ul class="simple"> |
| <li><p>Split the data into groups by using <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.groupBy.html#pyspark.sql.DataFrame.groupBy" title="pyspark.sql.DataFrame.groupBy"><code class="xref py py-meth docutils literal notranslate"><span class="pre">DataFrame.groupBy()</span></code></a>.</p></li> |
| <li><p>Apply a function on each group. The input and output of the function are both <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>. The input data contains all the rows and columns for each group.</p></li> |
| <li><p>Combine the results into a new PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a>.</p></li> |
| </ul> |
| <p>To use <code class="docutils literal notranslate"><span class="pre">DataFrame.groupBy().applyInPandas()</span></code>, the user needs to define the following:</p> |
| <ul class="simple"> |
| <li><p>A Python function that defines the computation for each group.</p></li> |
| <li><p>A <code class="docutils literal notranslate"><span class="pre">StructType</span></code> object or a string that defines the schema of the output PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a>.</p></li> |
| </ul> |
| <p>The column labels of the returned <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> must either match the field names in the |
| defined output schema if specified as strings, or match the field data types by position if not |
| strings, e.g. integer indices. See <a class="reference external" href="https://pandas.pydata.org/pandas-docs/stable/generated/pandas.DataFrame.html#pandas.DataFrame">pandas.DataFrame</a> |
| on how to label columns when constructing a <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>.</p> |
| <p>Note that all data for a group will be loaded into memory before the function is applied. This can |
| lead to out of memory exceptions, especially if the group sizes are skewed. The configuration for |
| <a class="reference internal" href="#setting-arrow-batch-size"><span class="std std-ref">maxRecordsPerBatch</span></a> is not applied on groups and it is up to the user |
| to ensure that the grouped data will fit into the available memory.</p> |
| <p>The following example shows how to use <code class="docutils literal notranslate"><span class="pre">DataFrame.groupby().applyInPandas()</span></code> to subtract the mean from each value |
| in the group.</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span> |
| <span class="p">[(</span><span class="mi">1</span><span class="p">,</span> <span class="mf">1.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">1</span><span class="p">,</span> <span class="mf">2.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">2</span><span class="p">,</span> <span class="mf">3.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">2</span><span class="p">,</span> <span class="mf">5.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">2</span><span class="p">,</span> <span class="mf">10.0</span><span class="p">)],</span> |
| <span class="p">(</span><span class="s2">"id"</span><span class="p">,</span> <span class="s2">"v"</span><span class="p">))</span> |
| |
| <span class="k">def</span> <span class="nf">subtract_mean</span><span class="p">(</span><span class="n">pdf</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">)</span> <span class="o">-></span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">:</span> |
| <span class="c1"># pdf is a pandas.DataFrame</span> |
| <span class="n">v</span> <span class="o">=</span> <span class="n">pdf</span><span class="o">.</span><span class="n">v</span> |
| <span class="k">return</span> <span class="n">pdf</span><span class="o">.</span><span class="n">assign</span><span class="p">(</span><span class="n">v</span><span class="o">=</span><span class="n">v</span> <span class="o">-</span> <span class="n">v</span><span class="o">.</span><span class="n">mean</span><span class="p">())</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">groupby</span><span class="p">(</span><span class="s2">"id"</span><span class="p">)</span><span class="o">.</span><span class="n">applyInPandas</span><span class="p">(</span><span class="n">subtract_mean</span><span class="p">,</span> <span class="n">schema</span><span class="o">=</span><span class="s2">"id long, v double"</span><span class="p">)</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +---+----+</span> |
| <span class="c1"># | id| v|</span> |
| <span class="c1"># +---+----+</span> |
| <span class="c1"># | 1|-0.5|</span> |
| <span class="c1"># | 1| 0.5|</span> |
| <span class="c1"># | 2|-3.0|</span> |
| <span class="c1"># | 2|-1.0|</span> |
| <span class="c1"># | 2| 4.0|</span> |
| <span class="c1"># +---+----+</span> |
| </pre></div> |
| </div> |
| <p>For detailed usage, please see please see <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.GroupedData.applyInPandas.html#pyspark.sql.GroupedData.applyInPandas" title="pyspark.sql.GroupedData.applyInPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">GroupedData.applyInPandas()</span></code></a></p> |
| </div> |
| <div class="section" id="map"> |
| <h3>Map<a class="headerlink" href="#map" title="Permalink to this headline">¶</a></h3> |
| <p>Map operations with Pandas instances are supported by <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.mapInPandas.html#pyspark.sql.DataFrame.mapInPandas" title="pyspark.sql.DataFrame.mapInPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">DataFrame.mapInPandas()</span></code></a> which maps an iterator |
| of <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>s to another iterator of <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>s that represents the current |
| PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a> and returns the result as a PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a>. The function takes and outputs |
| an iterator of <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>. It can return the output of arbitrary length in contrast to some |
| Pandas UDFs although internally it works similarly with Series to Series Pandas UDF.</p> |
| <p>The following example shows how to use <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.mapInPandas.html#pyspark.sql.DataFrame.mapInPandas" title="pyspark.sql.DataFrame.mapInPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">DataFrame.mapInPandas()</span></code></a>:</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="n">df</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">([(</span><span class="mi">1</span><span class="p">,</span> <span class="mi">21</span><span class="p">),</span> <span class="p">(</span><span class="mi">2</span><span class="p">,</span> <span class="mi">30</span><span class="p">)],</span> <span class="p">(</span><span class="s2">"id"</span><span class="p">,</span> <span class="s2">"age"</span><span class="p">))</span> |
| |
| <span class="k">def</span> <span class="nf">filter_func</span><span class="p">(</span><span class="n">iterator</span><span class="p">:</span> <span class="n">Iterable</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">])</span> <span class="o">-></span> <span class="n">Iterable</span><span class="p">[</span><span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">]:</span> |
| <span class="k">for</span> <span class="n">pdf</span> <span class="ow">in</span> <span class="n">iterator</span><span class="p">:</span> |
| <span class="k">yield</span> <span class="n">pdf</span><span class="p">[</span><span class="n">pdf</span><span class="o">.</span><span class="n">id</span> <span class="o">==</span> <span class="mi">1</span><span class="p">]</span> |
| |
| <span class="n">df</span><span class="o">.</span><span class="n">mapInPandas</span><span class="p">(</span><span class="n">filter_func</span><span class="p">,</span> <span class="n">schema</span><span class="o">=</span><span class="n">df</span><span class="o">.</span><span class="n">schema</span><span class="p">)</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +---+---+</span> |
| <span class="c1"># | id|age|</span> |
| <span class="c1"># +---+---+</span> |
| <span class="c1"># | 1| 21|</span> |
| <span class="c1"># +---+---+</span> |
| </pre></div> |
| </div> |
| <p>For detailed usage, please see <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.mapInPandas.html#pyspark.sql.DataFrame.mapInPandas" title="pyspark.sql.DataFrame.mapInPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">DataFrame.mapInPandas()</span></code></a>.</p> |
| </div> |
| <div class="section" id="co-grouped-map"> |
| <h3>Co-grouped Map<a class="headerlink" href="#co-grouped-map" title="Permalink to this headline">¶</a></h3> |
| <p>Co-grouped map operations with Pandas instances are supported by <code class="docutils literal notranslate"><span class="pre">DataFrame.groupby().cogroup().applyInPandas()</span></code> which |
| allows two PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a>s to be cogrouped by a common key and then a Python function applied to each |
| cogroup. It consists of the following steps:</p> |
| <ul class="simple"> |
| <li><p>Shuffle the data such that the groups of each dataframe which share a key are cogrouped together.</p></li> |
| <li><p>Apply a function to each cogroup. The input of the function is two <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> (with an optional tuple representing the key). The output of the function is a <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>.</p></li> |
| <li><p>Combine the <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>s from all groups into a new PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a>.</p></li> |
| </ul> |
| <p>To use <code class="docutils literal notranslate"><span class="pre">groupBy().cogroup().applyInPandas()</span></code>, the user needs to define the following:</p> |
| <ul class="simple"> |
| <li><p>A Python function that defines the computation for each cogroup.</p></li> |
| <li><p>A <code class="docutils literal notranslate"><span class="pre">StructType</span></code> object or a string that defines the schema of the output PySpark <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.html#pyspark.sql.DataFrame" title="pyspark.sql.DataFrame"><code class="xref py py-class docutils literal notranslate"><span class="pre">DataFrame</span></code></a>.</p></li> |
| </ul> |
| <p>The column labels of the returned <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code> must either match the field names in the |
| defined output schema if specified as strings, or match the field data types by position if not |
| strings, e.g. integer indices. See <a class="reference external" href="https://pandas.pydata.org/pandas-docs/stable/generated/pandas.DataFrame.html#pandas.DataFrame">pandas.DataFrame</a>. |
| on how to label columns when constructing a <code class="docutils literal notranslate"><span class="pre">pandas.DataFrame</span></code>.</p> |
| <p>Note that all data for a cogroup will be loaded into memory before the function is applied. This can lead to out of |
| memory exceptions, especially if the group sizes are skewed. The configuration for <a class="reference internal" href="#setting-arrow-batch-size"><span class="std std-ref">maxRecordsPerBatch</span></a> |
| is not applied and it is up to the user to ensure that the cogrouped data will fit into the available memory.</p> |
| <p>The following example shows how to use <code class="docutils literal notranslate"><span class="pre">DataFrame.groupby().cogroup().applyInPandas()</span></code> to perform an asof join between two datasets.</p> |
| <div class="highlight-python notranslate"><div class="highlight"><pre><span></span><span class="kn">import</span> <span class="nn">pandas</span> <span class="k">as</span> <span class="nn">pd</span> |
| |
| <span class="n">df1</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span> |
| <span class="p">[(</span><span class="mi">20000101</span><span class="p">,</span> <span class="mi">1</span><span class="p">,</span> <span class="mf">1.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">20000101</span><span class="p">,</span> <span class="mi">2</span><span class="p">,</span> <span class="mf">2.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">20000102</span><span class="p">,</span> <span class="mi">1</span><span class="p">,</span> <span class="mf">3.0</span><span class="p">),</span> <span class="p">(</span><span class="mi">20000102</span><span class="p">,</span> <span class="mi">2</span><span class="p">,</span> <span class="mf">4.0</span><span class="p">)],</span> |
| <span class="p">(</span><span class="s2">"time"</span><span class="p">,</span> <span class="s2">"id"</span><span class="p">,</span> <span class="s2">"v1"</span><span class="p">))</span> |
| |
| <span class="n">df2</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">createDataFrame</span><span class="p">(</span> |
| <span class="p">[(</span><span class="mi">20000101</span><span class="p">,</span> <span class="mi">1</span><span class="p">,</span> <span class="s2">"x"</span><span class="p">),</span> <span class="p">(</span><span class="mi">20000101</span><span class="p">,</span> <span class="mi">2</span><span class="p">,</span> <span class="s2">"y"</span><span class="p">)],</span> |
| <span class="p">(</span><span class="s2">"time"</span><span class="p">,</span> <span class="s2">"id"</span><span class="p">,</span> <span class="s2">"v2"</span><span class="p">))</span> |
| |
| <span class="k">def</span> <span class="nf">asof_join</span><span class="p">(</span><span class="n">left</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">,</span> <span class="n">right</span><span class="p">:</span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">)</span> <span class="o">-></span> <span class="n">pd</span><span class="o">.</span><span class="n">DataFrame</span><span class="p">:</span> |
| <span class="k">return</span> <span class="n">pd</span><span class="o">.</span><span class="n">merge_asof</span><span class="p">(</span><span class="n">left</span><span class="p">,</span> <span class="n">right</span><span class="p">,</span> <span class="n">on</span><span class="o">=</span><span class="s2">"time"</span><span class="p">,</span> <span class="n">by</span><span class="o">=</span><span class="s2">"id"</span><span class="p">)</span> |
| |
| <span class="n">df1</span><span class="o">.</span><span class="n">groupby</span><span class="p">(</span><span class="s2">"id"</span><span class="p">)</span><span class="o">.</span><span class="n">cogroup</span><span class="p">(</span><span class="n">df2</span><span class="o">.</span><span class="n">groupby</span><span class="p">(</span><span class="s2">"id"</span><span class="p">))</span><span class="o">.</span><span class="n">applyInPandas</span><span class="p">(</span> |
| <span class="n">asof_join</span><span class="p">,</span> <span class="n">schema</span><span class="o">=</span><span class="s2">"time int, id int, v1 double, v2 string"</span><span class="p">)</span><span class="o">.</span><span class="n">show</span><span class="p">()</span> |
| <span class="c1"># +--------+---+---+---+</span> |
| <span class="c1"># | time| id| v1| v2|</span> |
| <span class="c1"># +--------+---+---+---+</span> |
| <span class="c1"># |20000101| 1|1.0| x|</span> |
| <span class="c1"># |20000102| 1|3.0| x|</span> |
| <span class="c1"># |20000101| 2|2.0| y|</span> |
| <span class="c1"># |20000102| 2|4.0| y|</span> |
| <span class="c1"># +--------+---+---+---+</span> |
| </pre></div> |
| </div> |
| <p>For detailed usage, please see <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.PandasCogroupedOps.applyInPandas.html#pyspark.sql.PandasCogroupedOps.applyInPandas" title="pyspark.sql.PandasCogroupedOps.applyInPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">PandasCogroupedOps.applyInPandas()</span></code></a></p> |
| </div> |
| </div> |
| <div class="section" id="usage-notes"> |
| <h2>Usage Notes<a class="headerlink" href="#usage-notes" title="Permalink to this headline">¶</a></h2> |
| <div class="section" id="supported-sql-types"> |
| <h3>Supported SQL Types<a class="headerlink" href="#supported-sql-types" title="Permalink to this headline">¶</a></h3> |
| <p>Currently, all Spark SQL data types are supported by Arrow-based conversion except |
| <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.types.ArrayType.html#pyspark.sql.types.ArrayType" title="pyspark.sql.types.ArrayType"><code class="xref py py-class docutils literal notranslate"><span class="pre">ArrayType</span></code></a> of <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.types.TimestampType.html#pyspark.sql.types.TimestampType" title="pyspark.sql.types.TimestampType"><code class="xref py py-class docutils literal notranslate"><span class="pre">TimestampType</span></code></a>, and nested <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.types.StructType.html#pyspark.sql.types.StructType" title="pyspark.sql.types.StructType"><code class="xref py py-class docutils literal notranslate"><span class="pre">StructType</span></code></a>. |
| <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.types.MapType.html#pyspark.sql.types.MapType" title="pyspark.sql.types.MapType"><code class="xref py py-class docutils literal notranslate"><span class="pre">MapType</span></code></a> is only supported when using PyArrow 2.0.0 and above.</p> |
| </div> |
| <div class="section" id="setting-arrow-batch-size"> |
| <h3>Setting Arrow Batch Size<a class="headerlink" href="#setting-arrow-batch-size" title="Permalink to this headline">¶</a></h3> |
| <p>Data partitions in Spark are converted into Arrow record batches, which can temporarily lead to |
| high memory usage in the JVM. To avoid possible out of memory exceptions, the size of the Arrow |
| record batches can be adjusted by setting the conf <code class="docutils literal notranslate"><span class="pre">spark.sql.execution.arrow.maxRecordsPerBatch</span></code> |
| to an integer that will determine the maximum number of rows for each batch. The default value is |
| 10,000 records per batch. If the number of columns is large, the value should be adjusted |
| accordingly. Using this limit, each data partition will be made into 1 or more record batches for |
| processing.</p> |
| </div> |
| <div class="section" id="timestamp-with-time-zone-semantics"> |
| <h3>Timestamp with Time Zone Semantics<a class="headerlink" href="#timestamp-with-time-zone-semantics" title="Permalink to this headline">¶</a></h3> |
| <p>Spark internally stores timestamps as UTC values, and timestamp data that is brought in without |
| a specified time zone is converted as local time to UTC with microsecond resolution. When timestamp |
| data is exported or displayed in Spark, the session time zone is used to localize the timestamp |
| values. The session time zone is set with the configuration <code class="docutils literal notranslate"><span class="pre">spark.sql.session.timeZone</span></code> and will |
| default to the JVM system local time zone if not set. Pandas uses a <code class="docutils literal notranslate"><span class="pre">datetime64</span></code> type with nanosecond |
| resolution, <code class="docutils literal notranslate"><span class="pre">datetime64[ns]</span></code>, with optional time zone on a per-column basis.</p> |
| <p>When timestamp data is transferred from Spark to Pandas it will be converted to nanoseconds |
| and each column will be converted to the Spark session time zone then localized to that time |
| zone, which removes the time zone and displays values as local time. This will occur |
| when calling <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.DataFrame.toPandas.html#pyspark.sql.DataFrame.toPandas" title="pyspark.sql.DataFrame.toPandas"><code class="xref py py-meth docutils literal notranslate"><span class="pre">DataFrame.toPandas()</span></code></a> or <code class="docutils literal notranslate"><span class="pre">pandas_udf</span></code> with timestamp columns.</p> |
| <p>When timestamp data is transferred from Pandas to Spark, it will be converted to UTC microseconds. This |
| occurs when calling <a class="reference internal" href="../../reference/pyspark.sql/api/pyspark.sql.SparkSession.createDataFrame.html#pyspark.sql.SparkSession.createDataFrame" title="pyspark.sql.SparkSession.createDataFrame"><code class="xref py py-meth docutils literal notranslate"><span class="pre">SparkSession.createDataFrame()</span></code></a> with a Pandas DataFrame or when returning a timestamp from a |
| <code class="docutils literal notranslate"><span class="pre">pandas_udf</span></code>. These conversions are done automatically to ensure Spark will have data in the |
| expected format, so it is not necessary to do any of these conversions yourself. Any nanosecond |
| values will be truncated.</p> |
| <p>Note that a standard UDF (non-Pandas) will load timestamp data as Python datetime objects, which is |
| different than a Pandas timestamp. It is recommended to use Pandas time series functionality when |
| working with timestamps in <code class="docutils literal notranslate"><span class="pre">pandas_udf</span></code>s to get the best performance, see |
| <a class="reference external" href="https://pandas.pydata.org/pandas-docs/stable/timeseries.html">here</a> for details.</p> |
| </div> |
| <div class="section" id="recommended-pandas-and-pyarrow-versions"> |
| <h3>Recommended Pandas and PyArrow Versions<a class="headerlink" href="#recommended-pandas-and-pyarrow-versions" title="Permalink to this headline">¶</a></h3> |
| <p>For usage with pyspark.sql, the minimum supported versions of Pandas is 1.0.5 and PyArrow is 1.0.0. |
| Higher versions may be used, however, compatibility and data correctness can not be guaranteed and should |
| be verified by the user.</p> |
| </div> |
| <div class="section" id="setting-arrow-self-destruct-for-memory-savings"> |
| <h3>Setting Arrow <code class="docutils literal notranslate"><span class="pre">self_destruct</span></code> for memory savings<a class="headerlink" href="#setting-arrow-self-destruct-for-memory-savings" title="Permalink to this headline">¶</a></h3> |
| <p>Since Spark 3.2, the Spark configuration <code class="docutils literal notranslate"><span class="pre">spark.sql.execution.arrow.pyspark.selfDestruct.enabled</span></code> can be used to enable PyArrow’s <code class="docutils literal notranslate"><span class="pre">self_destruct</span></code> feature, which can save memory when creating a Pandas DataFrame via <code class="docutils literal notranslate"><span class="pre">toPandas</span></code> by freeing Arrow-allocated memory while building the Pandas DataFrame. |
| This option is experimental, and some operations may fail on the resulting Pandas DataFrame due to immutable backing arrays. |
| Typically, you would see the error <code class="docutils literal notranslate"><span class="pre">ValueError:</span> <span class="pre">buffer</span> <span class="pre">source</span> <span class="pre">array</span> <span class="pre">is</span> <span class="pre">read-only</span></code>. |
| Newer versions of Pandas may fix these errors by improving support for such cases. |
| You can work around this error by copying the column(s) beforehand. |
| Additionally, this conversion may be slower because it is single-threaded.</p> |
| </div> |
| </div> |
| </div> |
| |
| |
| </div> |
| |
| |
| <div class='prev-next-bottom'> |
| |
| <a class='left-prev' id="prev-link" href="index.html" title="previous page">Spark SQL</a> |
| <a class='right-next' id="next-link" href="../pandas_on_spark/index.html" title="next page">Pandas API on Spark</a> |
| |
| </div> |
| |
| </main> |
| |
| |
| </div> |
| </div> |
| |
| |
| <script src="../../_static/js/index.3da636dd464baa7582d2.js"></script> |
| |
| |
| <footer class="footer mt-5 mt-md-0"> |
| <div class="container"> |
| <p> |
| © Copyright .<br/> |
| Created using <a href="http://sphinx-doc.org/">Sphinx</a> 3.0.4.<br/> |
| </p> |
| </div> |
| </footer> |
| </body> |
| </html> |