blob: bd3d4835af79c51cff1cde4040aa2bf7b7a5f80c [file] [log] [blame]
<!DOCTYPE html>
<!--[if lt IE 7]> <html class="no-js lt-ie9 lt-ie8 lt-ie7"> <![endif]-->
<!--[if IE 7]> <html class="no-js lt-ie9 lt-ie8"> <![endif]-->
<!--[if IE 8]> <html class="no-js lt-ie9"> <![endif]-->
<!--[if gt IE 8]><!--> <html class="no-js"> <!--<![endif]-->
<head>
<meta charset="utf-8">
<meta http-equiv="X-UA-Compatible" content="IE=edge,chrome=1">
<title>Generic Load/Save Functions - Spark 2.4.4 Documentation</title>
<link rel="stylesheet" href="css/bootstrap.min.css">
<style>
body {
padding-top: 60px;
padding-bottom: 40px;
}
</style>
<meta name="viewport" content="width=device-width">
<link rel="stylesheet" href="css/bootstrap-responsive.min.css">
<link rel="stylesheet" href="css/main.css">
<script src="js/vendor/modernizr-2.6.1-respond-1.1.0.min.js"></script>
<link rel="stylesheet" href="css/pygments-default.css">
<!-- Google analytics script -->
<script type="text/javascript">
var _gaq = _gaq || [];
_gaq.push(['_setAccount', 'UA-32518208-2']);
_gaq.push(['_trackPageview']);
(function() {
var ga = document.createElement('script'); ga.type = 'text/javascript'; ga.async = true;
ga.src = ('https:' == document.location.protocol ? 'https://ssl' : 'http://www') + '.google-analytics.com/ga.js';
var s = document.getElementsByTagName('script')[0]; s.parentNode.insertBefore(ga, s);
})();
</script>
</head>
<body>
<!--[if lt IE 7]>
<p class="chromeframe">You are using an outdated browser. <a href="https://browsehappy.com/">Upgrade your browser today</a> or <a href="http://www.google.com/chromeframe/?redirect=true">install Google Chrome Frame</a> to better experience this site.</p>
<![endif]-->
<!-- This code is taken from http://twitter.github.com/bootstrap/examples/hero.html -->
<div class="navbar navbar-fixed-top" id="topbar">
<div class="navbar-inner">
<div class="container">
<div class="brand"><a href="index.html">
<img src="img/spark-logo-hd.png" style="height:50px;"/></a><span class="version">2.4.4</span>
</div>
<ul class="nav">
<!--TODO(andyk): Add class="active" attribute to li some how.-->
<li><a href="index.html">Overview</a></li>
<li class="dropdown">
<a href="#" class="dropdown-toggle" data-toggle="dropdown">Programming Guides<b class="caret"></b></a>
<ul class="dropdown-menu">
<li><a href="quick-start.html">Quick Start</a></li>
<li><a href="rdd-programming-guide.html">RDDs, Accumulators, Broadcasts Vars</a></li>
<li><a href="sql-programming-guide.html">SQL, DataFrames, and Datasets</a></li>
<li><a href="structured-streaming-programming-guide.html">Structured Streaming</a></li>
<li><a href="streaming-programming-guide.html">Spark Streaming (DStreams)</a></li>
<li><a href="ml-guide.html">MLlib (Machine Learning)</a></li>
<li><a href="graphx-programming-guide.html">GraphX (Graph Processing)</a></li>
<li><a href="sparkr.html">SparkR (R on Spark)</a></li>
</ul>
</li>
<li class="dropdown">
<a href="#" class="dropdown-toggle" data-toggle="dropdown">API Docs<b class="caret"></b></a>
<ul class="dropdown-menu">
<li><a href="api/scala/index.html#org.apache.spark.package">Scala</a></li>
<li><a href="api/java/index.html">Java</a></li>
<li><a href="api/python/index.html">Python</a></li>
<li><a href="api/R/index.html">R</a></li>
<li><a href="api/sql/index.html">SQL, Built-in Functions</a></li>
</ul>
</li>
<li class="dropdown">
<a href="#" class="dropdown-toggle" data-toggle="dropdown">Deploying<b class="caret"></b></a>
<ul class="dropdown-menu">
<li><a href="cluster-overview.html">Overview</a></li>
<li><a href="submitting-applications.html">Submitting Applications</a></li>
<li class="divider"></li>
<li><a href="spark-standalone.html">Spark Standalone</a></li>
<li><a href="running-on-mesos.html">Mesos</a></li>
<li><a href="running-on-yarn.html">YARN</a></li>
<li><a href="running-on-kubernetes.html">Kubernetes</a></li>
</ul>
</li>
<li class="dropdown">
<a href="api.html" class="dropdown-toggle" data-toggle="dropdown">More<b class="caret"></b></a>
<ul class="dropdown-menu">
<li><a href="configuration.html">Configuration</a></li>
<li><a href="monitoring.html">Monitoring</a></li>
<li><a href="tuning.html">Tuning Guide</a></li>
<li><a href="job-scheduling.html">Job Scheduling</a></li>
<li><a href="security.html">Security</a></li>
<li><a href="hardware-provisioning.html">Hardware Provisioning</a></li>
<li class="divider"></li>
<li><a href="building-spark.html">Building Spark</a></li>
<li><a href="https://spark.apache.org/contributing.html">Contributing to Spark</a></li>
<li><a href="https://spark.apache.org/third-party-projects.html">Third Party Projects</a></li>
</ul>
</li>
</ul>
<!--<p class="navbar-text pull-right"><span class="version-text">v2.4.4</span></p>-->
</div>
</div>
</div>
<div class="container-wrapper">
<div class="left-menu-wrapper">
<div class="left-menu">
<h3><a href="sql-programming-guide.html">Spark SQL Guide</a></h3>
<ul>
<li>
<a href="sql-getting-started.html">
Getting Started
</a>
</li>
<li>
<a href="sql-data-sources.html">
Data Sources
</a>
</li>
<ul>
<li>
<a href="sql-data-sources-load-save-functions.html">
<b>Generic Load/Save Functions</b>
</a>
</li>
<li>
<a href="sql-data-sources-parquet.html">
Parquet Files
</a>
</li>
<li>
<a href="sql-data-sources-orc.html">
ORC Files
</a>
</li>
<li>
<a href="sql-data-sources-json.html">
JSON Files
</a>
</li>
<li>
<a href="sql-data-sources-hive-tables.html">
Hive Tables
</a>
</li>
<li>
<a href="sql-data-sources-jdbc.html">
JDBC To Other Databases
</a>
</li>
<li>
<a href="sql-data-sources-avro.html">
Avro Files
</a>
</li>
<li>
<a href="sql-data-sources-troubleshooting.html">
Troubleshooting
</a>
</li>
</ul>
<li>
<a href="sql-performance-tuning.html">
Performance Tuning
</a>
</li>
<li>
<a href="sql-distributed-sql-engine.html">
Distributed SQL Engine
</a>
</li>
<li>
<a href="sql-pyspark-pandas-with-arrow.html">
PySpark Usage Guide for Pandas with Apache Arrow
</a>
</li>
<li>
<a href="sql-migration-guide.html">
Migration Guide
</a>
</li>
<li>
<a href="sql-reference.html">
Reference
</a>
</li>
</ul>
</div>
</div>
<input id="nav-trigger" class="nav-trigger" checked type="checkbox">
<label for="nav-trigger"></label>
<div class="content-with-sidebar" id="content">
<h1 class="title">Generic Load/Save Functions</h1>
<ul id="markdown-toc">
<li><a href="#manually-specifying-options" id="markdown-toc-manually-specifying-options">Manually Specifying Options</a></li>
<li><a href="#run-sql-on-files-directly" id="markdown-toc-run-sql-on-files-directly">Run SQL on files directly</a></li>
<li><a href="#save-modes" id="markdown-toc-save-modes">Save Modes</a></li>
<li><a href="#saving-to-persistent-tables" id="markdown-toc-saving-to-persistent-tables">Saving to Persistent Tables</a></li>
<li><a href="#bucketing-sorting-and-partitioning" id="markdown-toc-bucketing-sorting-and-partitioning">Bucketing, Sorting and Partitioning</a></li>
</ul>
<p>In the simplest form, the default data source (<code>parquet</code> unless otherwise configured by
<code>spark.sql.sources.default</code>) will be used for all operations.</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="k">val</span> <span class="n">usersDF</span> <span class="k">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">read</span><span class="o">.</span><span class="n">load</span><span class="o">(</span><span class="s">&quot;examples/src/main/resources/users.parquet&quot;</span><span class="o">)</span>
<span class="n">usersDF</span><span class="o">.</span><span class="n">select</span><span class="o">(</span><span class="s">&quot;name&quot;</span><span class="o">,</span> <span class="s">&quot;favorite_color&quot;</span><span class="o">).</span><span class="n">write</span><span class="o">.</span><span class="n">save</span><span class="o">(</span><span class="s">&quot;namesAndFavColors.parquet&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">Dataset</span><span class="o">&lt;</span><span class="n">Row</span><span class="o">&gt;</span> <span class="n">usersDF</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="na">read</span><span class="o">().</span><span class="na">load</span><span class="o">(</span><span class="s">&quot;examples/src/main/resources/users.parquet&quot;</span><span class="o">);</span>
<span class="n">usersDF</span><span class="o">.</span><span class="na">select</span><span class="o">(</span><span class="s">&quot;name&quot;</span><span class="o">,</span> <span class="s">&quot;favorite_color&quot;</span><span class="o">).</span><span class="na">write</span><span class="o">().</span><span class="na">save</span><span class="o">(</span><span class="s">&quot;namesAndFavColors.parquet&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<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">read</span><span class="o">.</span><span class="n">load</span><span class="p">(</span><span class="s2">&quot;examples/src/main/resources/users.parquet&quot;</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="s2">&quot;name&quot;</span><span class="p">,</span> <span class="s2">&quot;favorite_color&quot;</span><span class="p">)</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">save</span><span class="p">(</span><span class="s2">&quot;namesAndFavColors.parquet&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="r">
<div class="highlight"><pre><span></span>df <span class="o">&lt;-</span> read.df<span class="p">(</span><span class="s">&quot;examples/src/main/resources/users.parquet&quot;</span><span class="p">)</span>
write.df<span class="p">(</span>select<span class="p">(</span>df<span class="p">,</span> <span class="s">&quot;name&quot;</span><span class="p">,</span> <span class="s">&quot;favorite_color&quot;</span><span class="p">),</span> <span class="s">&quot;namesAndFavColors.parquet&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/r/RSparkSQLExample.R" in the Spark repo.</small></div>
</div>
</div>
<h3 id="manually-specifying-options">Manually Specifying Options</h3>
<p>You can also manually specify the data source that will be used along with any extra options
that you would like to pass to the data source. Data sources are specified by their fully qualified
name (i.e., <code>org.apache.spark.sql.parquet</code>), but for built-in sources you can also use their short
names (<code>json</code>, <code>parquet</code>, <code>jdbc</code>, <code>orc</code>, <code>libsvm</code>, <code>csv</code>, <code>text</code>). DataFrames loaded from any data
source type can be converted into other types using this syntax.</p>
<p>To load a JSON file you can use:</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="k">val</span> <span class="n">peopleDF</span> <span class="k">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">read</span><span class="o">.</span><span class="n">format</span><span class="o">(</span><span class="s">&quot;json&quot;</span><span class="o">).</span><span class="n">load</span><span class="o">(</span><span class="s">&quot;examples/src/main/resources/people.json&quot;</span><span class="o">)</span>
<span class="n">peopleDF</span><span class="o">.</span><span class="n">select</span><span class="o">(</span><span class="s">&quot;name&quot;</span><span class="o">,</span> <span class="s">&quot;age&quot;</span><span class="o">).</span><span class="n">write</span><span class="o">.</span><span class="n">format</span><span class="o">(</span><span class="s">&quot;parquet&quot;</span><span class="o">).</span><span class="n">save</span><span class="o">(</span><span class="s">&quot;namesAndAges.parquet&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">Dataset</span><span class="o">&lt;</span><span class="n">Row</span><span class="o">&gt;</span> <span class="n">peopleDF</span> <span class="o">=</span>
<span class="n">spark</span><span class="o">.</span><span class="na">read</span><span class="o">().</span><span class="na">format</span><span class="o">(</span><span class="s">&quot;json&quot;</span><span class="o">).</span><span class="na">load</span><span class="o">(</span><span class="s">&quot;examples/src/main/resources/people.json&quot;</span><span class="o">);</span>
<span class="n">peopleDF</span><span class="o">.</span><span class="na">select</span><span class="o">(</span><span class="s">&quot;name&quot;</span><span class="o">,</span> <span class="s">&quot;age&quot;</span><span class="o">).</span><span class="na">write</span><span class="o">().</span><span class="na">format</span><span class="o">(</span><span class="s">&quot;parquet&quot;</span><span class="o">).</span><span class="na">save</span><span class="o">(</span><span class="s">&quot;namesAndAges.parquet&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<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">read</span><span class="o">.</span><span class="n">load</span><span class="p">(</span><span class="s2">&quot;examples/src/main/resources/people.json&quot;</span><span class="p">,</span> <span class="n">format</span><span class="o">=</span><span class="s2">&quot;json&quot;</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="s2">&quot;name&quot;</span><span class="p">,</span> <span class="s2">&quot;age&quot;</span><span class="p">)</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">save</span><span class="p">(</span><span class="s2">&quot;namesAndAges.parquet&quot;</span><span class="p">,</span> <span class="n">format</span><span class="o">=</span><span class="s2">&quot;parquet&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="r">
<div class="highlight"><pre><span></span>df <span class="o">&lt;-</span> read.df<span class="p">(</span><span class="s">&quot;examples/src/main/resources/people.json&quot;</span><span class="p">,</span> <span class="s">&quot;json&quot;</span><span class="p">)</span>
namesAndAges <span class="o">&lt;-</span> select<span class="p">(</span>df<span class="p">,</span> <span class="s">&quot;name&quot;</span><span class="p">,</span> <span class="s">&quot;age&quot;</span><span class="p">)</span>
write.df<span class="p">(</span>namesAndAges<span class="p">,</span> <span class="s">&quot;namesAndAges.parquet&quot;</span><span class="p">,</span> <span class="s">&quot;parquet&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/r/RSparkSQLExample.R" in the Spark repo.</small></div>
</div>
</div>
<p>To load a CSV file you can use:</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="k">val</span> <span class="n">peopleDFCsv</span> <span class="k">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">read</span><span class="o">.</span><span class="n">format</span><span class="o">(</span><span class="s">&quot;csv&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">option</span><span class="o">(</span><span class="s">&quot;sep&quot;</span><span class="o">,</span> <span class="s">&quot;;&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">option</span><span class="o">(</span><span class="s">&quot;inferSchema&quot;</span><span class="o">,</span> <span class="s">&quot;true&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">option</span><span class="o">(</span><span class="s">&quot;header&quot;</span><span class="o">,</span> <span class="s">&quot;true&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">load</span><span class="o">(</span><span class="s">&quot;examples/src/main/resources/people.csv&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">Dataset</span><span class="o">&lt;</span><span class="n">Row</span><span class="o">&gt;</span> <span class="n">peopleDFCsv</span> <span class="o">=</span> <span class="n">spark</span><span class="o">.</span><span class="na">read</span><span class="o">().</span><span class="na">format</span><span class="o">(</span><span class="s">&quot;csv&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">option</span><span class="o">(</span><span class="s">&quot;sep&quot;</span><span class="o">,</span> <span class="s">&quot;;&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">option</span><span class="o">(</span><span class="s">&quot;inferSchema&quot;</span><span class="o">,</span> <span class="s">&quot;true&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">option</span><span class="o">(</span><span class="s">&quot;header&quot;</span><span class="o">,</span> <span class="s">&quot;true&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">load</span><span class="o">(</span><span class="s">&quot;examples/src/main/resources/people.csv&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<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">read</span><span class="o">.</span><span class="n">load</span><span class="p">(</span><span class="s2">&quot;examples/src/main/resources/people.csv&quot;</span><span class="p">,</span>
<span class="n">format</span><span class="o">=</span><span class="s2">&quot;csv&quot;</span><span class="p">,</span> <span class="n">sep</span><span class="o">=</span><span class="s2">&quot;:&quot;</span><span class="p">,</span> <span class="n">inferSchema</span><span class="o">=</span><span class="s2">&quot;true&quot;</span><span class="p">,</span> <span class="n">header</span><span class="o">=</span><span class="s2">&quot;true&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="r">
<div class="highlight"><pre><span></span>df <span class="o">&lt;-</span> read.df<span class="p">(</span><span class="s">&quot;examples/src/main/resources/people.csv&quot;</span><span class="p">,</span> <span class="s">&quot;csv&quot;</span><span class="p">,</span> sep <span class="o">=</span> <span class="s">&quot;;&quot;</span><span class="p">,</span> inferSchema <span class="o">=</span> <span class="kc">TRUE</span><span class="p">,</span> header <span class="o">=</span> <span class="kc">TRUE</span><span class="p">)</span>
namesAndAges <span class="o">&lt;-</span> select<span class="p">(</span>df<span class="p">,</span> <span class="s">&quot;name&quot;</span><span class="p">,</span> <span class="s">&quot;age&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/r/RSparkSQLExample.R" in the Spark repo.</small></div>
</div>
</div>
<p>The extra options are also used during write operation.
For example, you can control bloom filters and dictionary encodings for ORC data sources.
The following ORC example will create bloom filter on <code>favorite_color</code> and use dictionary encoding for <code>name</code> and <code>favorite_color</code>.
For Parquet, there exists <code>parquet.enable.dictionary</code>, too.
To find more detailed information about the extra ORC/Parquet options,
visit the official Apache ORC/Parquet websites.</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="n">usersDF</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">format</span><span class="o">(</span><span class="s">&quot;orc&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">option</span><span class="o">(</span><span class="s">&quot;orc.bloom.filter.columns&quot;</span><span class="o">,</span> <span class="s">&quot;favorite_color&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">option</span><span class="o">(</span><span class="s">&quot;orc.dictionary.key.threshold&quot;</span><span class="o">,</span> <span class="s">&quot;1.0&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">save</span><span class="o">(</span><span class="s">&quot;users_with_options.orc&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">usersDF</span><span class="o">.</span><span class="na">write</span><span class="o">().</span><span class="na">format</span><span class="o">(</span><span class="s">&quot;orc&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">option</span><span class="o">(</span><span class="s">&quot;orc.bloom.filter.columns&quot;</span><span class="o">,</span> <span class="s">&quot;favorite_color&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">option</span><span class="o">(</span><span class="s">&quot;orc.dictionary.key.threshold&quot;</span><span class="o">,</span> <span class="s">&quot;1.0&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">save</span><span class="o">(</span><span class="s">&quot;users_with_options.orc&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<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">read</span><span class="o">.</span><span class="n">orc</span><span class="p">(</span><span class="s2">&quot;examples/src/main/resources/users.orc&quot;</span><span class="p">)</span>
<span class="p">(</span><span class="n">df</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="s2">&quot;orc&quot;</span><span class="p">)</span>
<span class="o">.</span><span class="n">option</span><span class="p">(</span><span class="s2">&quot;orc.bloom.filter.columns&quot;</span><span class="p">,</span> <span class="s2">&quot;favorite_color&quot;</span><span class="p">)</span>
<span class="o">.</span><span class="n">option</span><span class="p">(</span><span class="s2">&quot;orc.dictionary.key.threshold&quot;</span><span class="p">,</span> <span class="s2">&quot;1.0&quot;</span><span class="p">)</span>
<span class="o">.</span><span class="n">save</span><span class="p">(</span><span class="s2">&quot;users_with_options.orc&quot;</span><span class="p">))</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="r">
<div class="highlight"><pre><span></span>df <span class="o">&lt;-</span> read.df<span class="p">(</span><span class="s">&quot;examples/src/main/resources/users.orc&quot;</span><span class="p">,</span> <span class="s">&quot;orc&quot;</span><span class="p">)</span>
write.orc<span class="p">(</span>df<span class="p">,</span> <span class="s">&quot;users_with_options.orc&quot;</span><span class="p">,</span> orc.bloom.filter.columns <span class="o">=</span> <span class="s">&quot;favorite_color&quot;</span><span class="p">,</span> orc.dictionary.key.threshold <span class="o">=</span> <span class="m">1.0</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/r/RSparkSQLExample.R" in the Spark repo.</small></div>
</div>
<div data-lang="sql">
<figure class="highlight"><pre><code class="language-sql" data-lang="sql"><span></span><span class="k">CREATE</span> <span class="k">TABLE</span> <span class="n">users_with_options</span> <span class="p">(</span>
<span class="n">name</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_color</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_numbers</span> <span class="nb">array</span><span class="o">&lt;</span><span class="nb">integer</span><span class="o">&gt;</span>
<span class="p">)</span> <span class="k">USING</span> <span class="n">ORC</span>
<span class="k">OPTIONS</span> <span class="p">(</span>
<span class="n">orc</span><span class="p">.</span><span class="n">bloom</span><span class="p">.</span><span class="n">filter</span><span class="p">.</span><span class="n">columns</span> <span class="s1">&#39;favorite_color&#39;</span><span class="p">,</span>
<span class="n">orc</span><span class="p">.</span><span class="k">dictionary</span><span class="p">.</span><span class="k">key</span><span class="p">.</span><span class="n">threshold</span> <span class="s1">&#39;1.0&#39;</span>
<span class="p">)</span></code></pre></figure>
</div>
</div>
<h3 id="run-sql-on-files-directly">Run SQL on files directly</h3>
<p>Instead of using read API to load a file into DataFrame and query it, you can also query that
file directly with SQL.</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="k">val</span> <span class="n">sqlDF</span> <span class="k">=</span> <span class="n">spark</span><span class="o">.</span><span class="n">sql</span><span class="o">(</span><span class="s">&quot;SELECT * FROM parquet.`examples/src/main/resources/users.parquet`&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">Dataset</span><span class="o">&lt;</span><span class="n">Row</span><span class="o">&gt;</span> <span class="n">sqlDF</span> <span class="o">=</span>
<span class="n">spark</span><span class="o">.</span><span class="na">sql</span><span class="o">(</span><span class="s">&quot;SELECT * FROM parquet.`examples/src/main/resources/users.parquet`&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<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">sql</span><span class="p">(</span><span class="s2">&quot;SELECT * FROM parquet.`examples/src/main/resources/users.parquet`&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="r">
<div class="highlight"><pre><span></span>df <span class="o">&lt;-</span> sql<span class="p">(</span><span class="s">&quot;SELECT * FROM parquet.`examples/src/main/resources/users.parquet`&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/r/RSparkSQLExample.R" in the Spark repo.</small></div>
</div>
</div>
<h3 id="save-modes">Save Modes</h3>
<p>Save operations can optionally take a <code>SaveMode</code>, that specifies how to handle existing data if
present. It is important to realize that these save modes do not utilize any locking and are not
atomic. Additionally, when performing an <code>Overwrite</code>, the data will be deleted before writing out the
new data.</p>
<table class="table">
<tr><th>Scala/Java</th><th>Any Language</th><th>Meaning</th></tr>
<tr>
<td><code>SaveMode.ErrorIfExists</code> (default)</td>
<td><code>"error" or "errorifexists"</code> (default)</td>
<td>
When saving a DataFrame to a data source, if data already exists,
an exception is expected to be thrown.
</td>
</tr>
<tr>
<td><code>SaveMode.Append</code></td>
<td><code>"append"</code></td>
<td>
When saving a DataFrame to a data source, if data/table already exists,
contents of the DataFrame are expected to be appended to existing data.
</td>
</tr>
<tr>
<td><code>SaveMode.Overwrite</code></td>
<td><code>"overwrite"</code></td>
<td>
Overwrite mode means that when saving a DataFrame to a data source,
if data/table already exists, existing data is expected to be overwritten by the contents of
the DataFrame.
</td>
</tr>
<tr>
<td><code>SaveMode.Ignore</code></td>
<td><code>"ignore"</code></td>
<td>
Ignore mode means that when saving a DataFrame to a data source, if data already exists,
the save operation is expected not to save the contents of the DataFrame and not to
change the existing data. This is similar to a <code>CREATE TABLE IF NOT EXISTS</code> in SQL.
</td>
</tr>
</table>
<h3 id="saving-to-persistent-tables">Saving to Persistent Tables</h3>
<p><code>DataFrames</code> can also be saved as persistent tables into Hive metastore using the <code>saveAsTable</code>
command. Notice that an existing Hive deployment is not necessary to use this feature. Spark will create a
default local Hive metastore (using Derby) for you. Unlike the <code>createOrReplaceTempView</code> command,
<code>saveAsTable</code> will materialize the contents of the DataFrame and create a pointer to the data in the
Hive metastore. Persistent tables will still exist even after your Spark program has restarted, as
long as you maintain your connection to the same metastore. A DataFrame for a persistent table can
be created by calling the <code>table</code> method on a <code>SparkSession</code> with the name of the table.</p>
<p>For file-based data source, e.g. text, parquet, json, etc. you can specify a custom table path via the
<code>path</code> option, e.g. <code>df.write.option("path", "/some/path").saveAsTable("t")</code>. When the table is dropped,
the custom table path will not be removed and the table data is still there. If no custom table path is
specified, Spark will write data to a default table path under the warehouse directory. When the table is
dropped, the default table path will be removed too.</p>
<p>Starting from Spark 2.1, persistent datasource tables have per-partition metadata stored in the Hive metastore. This brings several benefits:</p>
<ul>
<li>Since the metastore can return only necessary partitions for a query, discovering all the partitions on the first query to the table is no longer needed.</li>
<li>Hive DDLs such as <code>ALTER TABLE PARTITION ... SET LOCATION</code> are now available for tables created with the Datasource API.</li>
</ul>
<p>Note that partition information is not gathered by default when creating external datasource tables (those with a <code>path</code> option). To sync the partition information in the metastore, you can invoke <code>MSCK REPAIR TABLE</code>.</p>
<h3 id="bucketing-sorting-and-partitioning">Bucketing, Sorting and Partitioning</h3>
<p>For file-based data source, it is also possible to bucket and sort or partition the output.
Bucketing and sorting are applicable only to persistent tables:</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="n">peopleDF</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">bucketBy</span><span class="o">(</span><span class="mi">42</span><span class="o">,</span> <span class="s">&quot;name&quot;</span><span class="o">).</span><span class="n">sortBy</span><span class="o">(</span><span class="s">&quot;age&quot;</span><span class="o">).</span><span class="n">saveAsTable</span><span class="o">(</span><span class="s">&quot;people_bucketed&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">peopleDF</span><span class="o">.</span><span class="na">write</span><span class="o">().</span><span class="na">bucketBy</span><span class="o">(</span><span class="mi">42</span><span class="o">,</span> <span class="s">&quot;name&quot;</span><span class="o">).</span><span class="na">sortBy</span><span class="o">(</span><span class="s">&quot;age&quot;</span><span class="o">).</span><span class="na">saveAsTable</span><span class="o">(</span><span class="s">&quot;people_bucketed&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<div class="highlight"><pre><span></span><span class="n">df</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">bucketBy</span><span class="p">(</span><span class="mi">42</span><span class="p">,</span> <span class="s2">&quot;name&quot;</span><span class="p">)</span><span class="o">.</span><span class="n">sortBy</span><span class="p">(</span><span class="s2">&quot;age&quot;</span><span class="p">)</span><span class="o">.</span><span class="n">saveAsTable</span><span class="p">(</span><span class="s2">&quot;people_bucketed&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="sql">
<figure class="highlight"><pre><code class="language-sql" data-lang="sql"><span></span><span class="k">CREATE</span> <span class="k">TABLE</span> <span class="n">users_bucketed_by_name</span><span class="p">(</span>
<span class="n">name</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_color</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_numbers</span> <span class="nb">array</span><span class="o">&lt;</span><span class="nb">integer</span><span class="o">&gt;</span>
<span class="p">)</span> <span class="k">USING</span> <span class="n">parquet</span>
<span class="n">CLUSTERED</span> <span class="k">BY</span><span class="p">(</span><span class="n">name</span><span class="p">)</span> <span class="k">INTO</span> <span class="mi">42</span> <span class="n">BUCKETS</span><span class="p">;</span></code></pre></figure>
</div>
</div>
<p>while partitioning can be used with both <code>save</code> and <code>saveAsTable</code> when using the Dataset APIs.</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="n">usersDF</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">partitionBy</span><span class="o">(</span><span class="s">&quot;favorite_color&quot;</span><span class="o">).</span><span class="n">format</span><span class="o">(</span><span class="s">&quot;parquet&quot;</span><span class="o">).</span><span class="n">save</span><span class="o">(</span><span class="s">&quot;namesPartByColor.parquet&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">usersDF</span>
<span class="o">.</span><span class="na">write</span><span class="o">()</span>
<span class="o">.</span><span class="na">partitionBy</span><span class="o">(</span><span class="s">&quot;favorite_color&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">format</span><span class="o">(</span><span class="s">&quot;parquet&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">save</span><span class="o">(</span><span class="s">&quot;namesPartByColor.parquet&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<div class="highlight"><pre><span></span><span class="n">df</span><span class="o">.</span><span class="n">write</span><span class="o">.</span><span class="n">partitionBy</span><span class="p">(</span><span class="s2">&quot;favorite_color&quot;</span><span class="p">)</span><span class="o">.</span><span class="n">format</span><span class="p">(</span><span class="s2">&quot;parquet&quot;</span><span class="p">)</span><span class="o">.</span><span class="n">save</span><span class="p">(</span><span class="s2">&quot;namesPartByColor.parquet&quot;</span><span class="p">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="sql">
<figure class="highlight"><pre><code class="language-sql" data-lang="sql"><span></span><span class="k">CREATE</span> <span class="k">TABLE</span> <span class="n">users_by_favorite_color</span><span class="p">(</span>
<span class="n">name</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_color</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_numbers</span> <span class="nb">array</span><span class="o">&lt;</span><span class="nb">integer</span><span class="o">&gt;</span>
<span class="p">)</span> <span class="k">USING</span> <span class="n">csv</span> <span class="n">PARTITIONED</span> <span class="k">BY</span><span class="p">(</span><span class="n">favorite_color</span><span class="p">);</span></code></pre></figure>
</div>
</div>
<p>It is possible to use both partitioning and bucketing for a single table:</p>
<div class="codetabs">
<div data-lang="scala">
<div class="highlight"><pre><span></span><span class="n">usersDF</span>
<span class="o">.</span><span class="n">write</span>
<span class="o">.</span><span class="n">partitionBy</span><span class="o">(</span><span class="s">&quot;favorite_color&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">bucketBy</span><span class="o">(</span><span class="mi">42</span><span class="o">,</span> <span class="s">&quot;name&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="n">saveAsTable</span><span class="o">(</span><span class="s">&quot;users_partitioned_bucketed&quot;</span><span class="o">)</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/scala/org/apache/spark/examples/sql/SQLDataSourceExample.scala" in the Spark repo.</small></div>
</div>
<div data-lang="java">
<div class="highlight"><pre><span></span><span class="n">peopleDF</span>
<span class="o">.</span><span class="na">write</span><span class="o">()</span>
<span class="o">.</span><span class="na">partitionBy</span><span class="o">(</span><span class="s">&quot;favorite_color&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">bucketBy</span><span class="o">(</span><span class="mi">42</span><span class="o">,</span> <span class="s">&quot;name&quot;</span><span class="o">)</span>
<span class="o">.</span><span class="na">saveAsTable</span><span class="o">(</span><span class="s">&quot;people_partitioned_bucketed&quot;</span><span class="o">);</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/java/org/apache/spark/examples/sql/JavaSQLDataSourceExample.java" in the Spark repo.</small></div>
</div>
<div data-lang="python">
<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">read</span><span class="o">.</span><span class="n">parquet</span><span class="p">(</span><span class="s2">&quot;examples/src/main/resources/users.parquet&quot;</span><span class="p">)</span>
<span class="p">(</span><span class="n">df</span>
<span class="o">.</span><span class="n">write</span>
<span class="o">.</span><span class="n">partitionBy</span><span class="p">(</span><span class="s2">&quot;favorite_color&quot;</span><span class="p">)</span>
<span class="o">.</span><span class="n">bucketBy</span><span class="p">(</span><span class="mi">42</span><span class="p">,</span> <span class="s2">&quot;name&quot;</span><span class="p">)</span>
<span class="o">.</span><span class="n">saveAsTable</span><span class="p">(</span><span class="s2">&quot;people_partitioned_bucketed&quot;</span><span class="p">))</span>
</pre></div>
<div><small>Find full example code at "examples/src/main/python/sql/datasource.py" in the Spark repo.</small></div>
</div>
<div data-lang="sql">
<figure class="highlight"><pre><code class="language-sql" data-lang="sql"><span></span><span class="k">CREATE</span> <span class="k">TABLE</span> <span class="n">users_bucketed_and_partitioned</span><span class="p">(</span>
<span class="n">name</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_color</span> <span class="n">STRING</span><span class="p">,</span>
<span class="n">favorite_numbers</span> <span class="nb">array</span><span class="o">&lt;</span><span class="nb">integer</span><span class="o">&gt;</span>
<span class="p">)</span> <span class="k">USING</span> <span class="n">parquet</span>
<span class="n">PARTITIONED</span> <span class="k">BY</span> <span class="p">(</span><span class="n">favorite_color</span><span class="p">)</span>
<span class="n">CLUSTERED</span> <span class="k">BY</span><span class="p">(</span><span class="n">name</span><span class="p">)</span> <span class="n">SORTED</span> <span class="k">BY</span> <span class="p">(</span><span class="n">favorite_numbers</span><span class="p">)</span> <span class="k">INTO</span> <span class="mi">42</span> <span class="n">BUCKETS</span><span class="p">;</span></code></pre></figure>
</div>
</div>
<p><code>partitionBy</code> creates a directory structure as described in the <a href="sql-data-sources-parquet.html#partition-discovery">Partition Discovery</a> section.
Thus, it has limited applicability to columns with high cardinality. In contrast
<code>bucketBy</code> distributes
data across a fixed number of buckets and can be used when a number of unique values is unbounded.</p>
</div>
<!-- /container -->
</div>
<script src="js/vendor/jquery-1.12.4.min.js"></script>
<script src="js/vendor/bootstrap.min.js"></script>
<script src="js/vendor/anchor.min.js"></script>
<script src="js/main.js"></script>
<!-- MathJax Section -->
<script type="text/x-mathjax-config">
MathJax.Hub.Config({
TeX: { equationNumbers: { autoNumber: "AMS" } }
});
</script>
<script>
// Note that we load MathJax this way to work with local file (file://), HTTP and HTTPS.
// We could use "//cdn.mathjax...", but that won't support "file://".
(function(d, script) {
script = d.createElement('script');
script.type = 'text/javascript';
script.async = true;
script.onload = function(){
MathJax.Hub.Config({
tex2jax: {
inlineMath: [ ["$", "$"], ["\\\\(","\\\\)"] ],
displayMath: [ ["$$","$$"], ["\\[", "\\]"] ],
processEscapes: true,
skipTags: ['script', 'noscript', 'style', 'textarea', 'pre']
}
});
};
script.src = ('https:' == document.location.protocol ? 'https://' : 'http://') +
'cdnjs.cloudflare.com/ajax/libs/mathjax/2.7.1/MathJax.js' +
'?config=TeX-AMS-MML_HTMLorMML';
d.getElementsByTagName('head')[0].appendChild(script);
}(document));
</script>
</body>
</html>