aboutsummaryrefslogtreecommitdiff
path: root/examples/src/main/python
diff options
context:
space:
mode:
authorUri Laserson <laserson@cloudera.com>2014-09-27 21:48:05 -0700
committerMatei Zaharia <matei@databricks.com>2014-09-27 21:48:05 -0700
commit248232936e1bead7f102e59eb8faf3126c582d9d (patch)
tree8bee3f6fd2ba42230777a5a10ed47deb8f6863b1 /examples/src/main/python
parent5b922bb458e863f5be0ae68167de882743f70b86 (diff)
downloadspark-248232936e1bead7f102e59eb8faf3126c582d9d.tar.gz
spark-248232936e1bead7f102e59eb8faf3126c582d9d.tar.bz2
spark-248232936e1bead7f102e59eb8faf3126c582d9d.zip
[SPARK-3389] Add Converter for ease of Parquet reading in PySpark
https://issues.apache.org/jira/browse/SPARK-3389 Author: Uri Laserson <laserson@cloudera.com> Closes #2256 from laserson/SPARK-3389 and squashes the following commits: 0ed363e [Uri Laserson] PEP8'd the python file 0b4b380 [Uri Laserson] Moved converter to examples and added python example eecf4dc [Uri Laserson] [SPARK-3389] Add Converter for ease of Parquet reading in PySpark
Diffstat (limited to 'examples/src/main/python')
-rw-r--r--examples/src/main/python/parquet_inputformat.py59
1 files changed, 59 insertions, 0 deletions
diff --git a/examples/src/main/python/parquet_inputformat.py b/examples/src/main/python/parquet_inputformat.py
new file mode 100644
index 0000000000..c9b08f878a
--- /dev/null
+++ b/examples/src/main/python/parquet_inputformat.py
@@ -0,0 +1,59 @@
+#
+# Licensed to the Apache Software Foundation (ASF) under one or more
+# contributor license agreements. See the NOTICE file distributed with
+# this work for additional information regarding copyright ownership.
+# The ASF licenses this file to You under the Apache License, Version 2.0
+# (the "License"); you may not use this file except in compliance with
+# the License. You may obtain a copy of the License at
+#
+# http://www.apache.org/licenses/LICENSE-2.0
+#
+# Unless required by applicable law or agreed to in writing, software
+# distributed under the License is distributed on an "AS IS" BASIS,
+# WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
+# See the License for the specific language governing permissions and
+# limitations under the License.
+#
+
+import sys
+
+from pyspark import SparkContext
+
+"""
+Read data file users.parquet in local Spark distro:
+
+$ cd $SPARK_HOME
+$ export AVRO_PARQUET_JARS=/path/to/parquet-avro-1.5.0.jar
+$ ./bin/spark-submit --driver-class-path /path/to/example/jar \\
+ --jars $AVRO_PARQUET_JARS \\
+ ./examples/src/main/python/parquet_inputformat.py \\
+ examples/src/main/resources/users.parquet
+<...lots of log output...>
+{u'favorite_color': None, u'name': u'Alyssa', u'favorite_numbers': [3, 9, 15, 20]}
+{u'favorite_color': u'red', u'name': u'Ben', u'favorite_numbers': []}
+<...more log output...>
+"""
+if __name__ == "__main__":
+ if len(sys.argv) != 2:
+ print >> sys.stderr, """
+ Usage: parquet_inputformat.py <data_file>
+
+ Run with example jar:
+ ./bin/spark-submit --driver-class-path /path/to/example/jar \\
+ /path/to/examples/parquet_inputformat.py <data_file>
+ Assumes you have Parquet data stored in <data_file>.
+ """
+ exit(-1)
+
+ path = sys.argv[1]
+ sc = SparkContext(appName="ParquetInputFormat")
+
+ parquet_rdd = sc.newAPIHadoopFile(
+ path,
+ 'parquet.avro.AvroParquetInputFormat',
+ 'java.lang.Void',
+ 'org.apache.avro.generic.IndexedRecord',
+ valueConverter='org.apache.spark.examples.pythonconverters.IndexedRecordToJavaConverter')
+ output = parquet_rdd.map(lambda x: x[1]).collect()
+ for k in output:
+ print k