Skip to content

Latest commit

 

History

History
302 lines (279 loc) · 39.6 KB

configs.md

File metadata and controls

302 lines (279 loc) · 39.6 KB
layout title nav_order
page
Configuration
3

RAPIDS Accelerator for Apache Spark Configuration

The following is the list of options that rapids-plugin-4-spark supports.

On startup use: --conf [conf key]=[conf value]. For example:

${SPARK_HOME}/bin/spark --jars 'rapids-4-spark_2.12-0.2.0.jar,cudf-0.15-cuda10-1.jar' \
--conf spark.plugins=com.nvidia.spark.SQLPlugin \
--conf spark.rapids.sql.incompatibleOps.enabled=true

At runtime use: spark.conf.set("[conf key]", [conf value]). For example:

scala> spark.conf.set("spark.rapids.sql.incompatibleOps.enabled", true)

All configs can be set on startup, but some configs, especially for shuffle, will not work if they are set at runtime.

General Configuration

Name Description Default Value
spark.rapids.memory.gpu.allocFraction The fraction of total GPU memory that should be initially allocated for pooled memory. Extra memory will be allocated as needed, but it may result in more fragmentation. This must be less than or equal to the maximum limit configured via spark.rapids.memory.gpu.maxAllocFraction. 0.9
spark.rapids.memory.gpu.debug Provides a log of GPU memory allocations and frees. If set to STDOUT or STDERR the logging will go there. Setting it to NONE disables logging. All other values are reserved for possible future expansion and in the mean time will disable logging. NONE
spark.rapids.memory.gpu.maxAllocFraction The fraction of total GPU memory that limits the maximum size of the RMM pool. The value must be greater than or equal to the setting for spark.rapids.memory.gpu.allocFraction. Note that this limit will be reduced by the reserve memory configured in spark.rapids.memory.gpu.reserve. 1.0
spark.rapids.memory.gpu.pooling.enabled Should RMM act as a pooling allocator for GPU memory, or should it just pass through to CUDA memory allocation directly. true
spark.rapids.memory.gpu.reserve The amount of GPU memory that should remain unallocated by RMM and left for system use such as memory needed for kernels, kernel launches or JIT compilation. 1073741824
spark.rapids.memory.host.spillStorageSize Amount of off-heap host memory to use for buffering spilled GPU data before spilling to local disk 1073741824
spark.rapids.memory.pinnedPool.size The size of the pinned memory pool in bytes unless otherwise specified. Use 0 to disable the pool. 0
spark.rapids.python.concurrentPythonWorkers Set the number of Python worker processes that can execute concurrently per GPU. Python worker processes may temporarily block when the number of concurrent Python worker processes started by the same executor exceeds this amount. Allowing too many concurrent tasks on the same GPU may lead to GPU out of memory errors. >0 means enabled, while <=0 means unlimited 0
spark.rapids.python.memory.gpu.allocFraction The fraction of total GPU memory that should be initially allocated for pooled memory for all the Python workers. It supposes to be less than (1 - $(spark.rapids.memory.gpu.allocFraction)), since the executor will share the GPU with its owning Python workers. Half of the rest will be used if not specified None
spark.rapids.python.memory.gpu.maxAllocFraction The fraction of total GPU memory that limits the maximum size of the RMM pool for all the Python workers. It supposes to be less than (1 - $(spark.rapids.memory.gpu.maxAllocFraction)), since the executor will share the GPU with its owning Python workers. when setting to 0 it means no limit. 0.0
spark.rapids.python.memory.gpu.pooling.enabled Should RMM in Python workers act as a pooling allocator for GPU memory, or should it just pass through to CUDA memory allocation directly. When not specified, It will honor the value of config 'spark.rapids.memory.gpu.pooling.enabled' None
spark.rapids.shuffle.transport.enabled When set to true, enable the Rapids Shuffle Transport for accelerated shuffle. false
spark.rapids.shuffle.transport.maxReceiveInflightBytes Maximum aggregate amount of bytes that be fetched at any given time from peers during shuffle 1073741824
spark.rapids.shuffle.ucx.managementServerHost The host to be used to start the management server null
spark.rapids.shuffle.ucx.useWakeup When set to true, use UCX's event-based progress (epoll) in order to wake up the progress thread when needed, instead of a hot loop. true
spark.rapids.sql.batchSizeBytes Set the target number of bytes for a GPU batch. Splits sizes for input data is covered by separate configs. 2147483647
spark.rapids.sql.castFloatToString.enabled Casting from floating point types to string on the GPU returns results that have a different precision than the default Java toString behavior. false
spark.rapids.sql.castStringToFloat.enabled When set to true, enables casting from strings to float types (float, double) on the GPU. Currently hex values aren't supported on the GPU. Also note that casting from string to float types on the GPU returns incorrect results when the string represents any number "1.7976931348623158E308" <= x < "1.7976931348623159E308" and "-1.7976931348623158E308" >= x > "-1.7976931348623159E308" in both these cases the GPU returns Double.MaxValue while CPU returns "+Infinity" and "-Infinity" respectively false
spark.rapids.sql.castStringToInteger.enabled When set to true, enables casting from strings to integer types (byte, short, int, long) on the GPU. Casting from string to integer types on the GPU returns incorrect results when the string represents a number larger than Long.MaxValue or smaller than Long.MinValue. false
spark.rapids.sql.castStringToTimestamp.enabled When set to true, casting from string to timestamp is supported on the GPU. The GPU only supports a subset of formats when casting strings to timestamps. Refer to the CAST documentation for more details. false
spark.rapids.sql.concurrentGpuTasks Set the number of tasks that can execute concurrently per GPU. Tasks may temporarily block when the number of concurrent tasks in the executor exceeds this amount. Allowing too many concurrent tasks on the same GPU may lead to GPU out of memory errors. 1
spark.rapids.sql.csvTimestamps.enabled When set to true, enables the CSV parser to read timestamps. The default output format for Spark includes a timezone at the end. Anything except the UTC timezone is not supported. Timestamps after 2038 and before 1902 are also not supported. false
spark.rapids.sql.enabled Enable (true) or disable (false) sql operations on the GPU true
spark.rapids.sql.explain Explain why some parts of a query were not placed on a GPU or not. Possible values are ALL: print everything, NONE: print nothing, NOT_ON_GPU: print only parts of a query that did not go on the GPU NONE
spark.rapids.sql.format.csv.enabled When set to false disables all csv input and output acceleration. (only input is currently supported anyways) true
spark.rapids.sql.format.csv.read.enabled When set to false disables csv input acceleration true
spark.rapids.sql.format.orc.enabled When set to false disables all orc input and output acceleration true
spark.rapids.sql.format.orc.read.enabled When set to false disables orc input acceleration true
spark.rapids.sql.format.orc.write.enabled When set to false disables orc output acceleration true
spark.rapids.sql.format.parquet.enabled When set to false disables all parquet input and output acceleration true
spark.rapids.sql.format.parquet.multiThreadedRead.enabled When set to true, reads multiple small files within a partition more efficiently by reading each file in a separate thread in parallel on the CPU side before sending to the GPU. Limited by spark.rapids.sql.format.parquet.multiThreadedRead.numThreads and spark.rapids.sql.format.parquet.multiThreadedRead.maxNumFileProcessed true
spark.rapids.sql.format.parquet.multiThreadedRead.maxNumFilesParallel A limit on the maximum number of files per task processed in parallel on the CPU side before the file is sent to the GPU. This affects the amount of host memory used when reading the files in parallel. 2147483647
spark.rapids.sql.format.parquet.multiThreadedRead.numThreads The maximum number of threads, on the executor, to use for reading small parquet files in parallel. This can not be changed at runtime after the executor has started. 20
spark.rapids.sql.format.parquet.read.enabled When set to false disables parquet input acceleration true
spark.rapids.sql.format.parquet.write.enabled When set to false disables parquet output acceleration true
spark.rapids.sql.hasNans Config to indicate if your data has NaN's. Cudf doesn't currently support NaN's properly so you can get corrupt data if you have NaN's in your data and it runs on the GPU. true
spark.rapids.sql.hashOptimizeSort.enabled Whether sorts should be inserted after some hashed operations to improve output ordering. This can improve output file sizes when saving to columnar formats. false
spark.rapids.sql.improvedFloatOps.enabled For some floating point operations spark uses one way to compute the value and the underlying cudf implementation can use an improved algorithm. In some cases this can result in cudf producing an answer when spark overflows. Because this is not as compatible with spark, we have it disabled by default. false
spark.rapids.sql.improvedTimeOps.enabled When set to true, some operators will avoid overflowing by converting epoch days directly to seconds without first converting to microseconds false
spark.rapids.sql.incompatibleOps.enabled For operations that work, but are not 100% compatible with the Spark equivalent set if they should be enabled by default or disabled by default. false
spark.rapids.sql.python.gpu.enabled This is an experimental feature and is likely to change in the future. Enable (true) or disable (false) support for scheduling Python Pandas UDFs with GPU resources. When enabled, pandas UDFs are assumed to share the same GPU that the RAPIDs accelerator uses and will honor the python GPU configs false
spark.rapids.sql.reader.batchSizeBytes Soft limit on the maximum number of bytes the reader reads per batch. The readers will read chunks of data until this limit is met or exceeded. Note that the reader may estimate the number of bytes that will be used on the GPU in some cases based on the schema and number of rows in each batch. 2147483647
spark.rapids.sql.reader.batchSizeRows Soft limit on the maximum number of rows the reader will read per batch. The orc and parquet readers will read row groups until this limit is met or exceeded. The limit is respected by the csv reader. 2147483647
spark.rapids.sql.replaceSortMergeJoin.enabled Allow replacing sortMergeJoin with HashJoin true
spark.rapids.sql.shuffle.spillThreads Number of threads used to spill shuffle data to disk in the background. 6
spark.rapids.sql.udfCompiler.enabled When set to true, Scala UDFs will be considered for compilation as Catalyst expressions false
spark.rapids.sql.variableFloatAgg.enabled Spark assumes that all operations produce the exact same result each time. This is not true for some floating point aggregations, which can produce slightly different results on the GPU as the aggregation is done in parallel. This can enable those operations if you know the query is only computing it once. false

Supported GPU Operators and Fine Tuning

The RAPIDS Accelerator for Apache Spark can be configured to enable or disable specific GPU accelerated expressions. Enabled expressions are candidates for GPU execution. If the expression is configured as disabled, the accelerator plugin will not attempt replacement, and it will run on the CPU.

Please leverage the spark.rapids.sql.explain setting to get feedback from the plugin as to why parts of a query may not be executing on the GPU.

NOTE: Setting spark.rapids.sql.incompatibleOps.enabled=true will enable all the settings in the table below which are not enabled by default due to incompatibilities.

Expressions

Name SQL Function(s) Description Default Value Notes
spark.rapids.sql.expression.Abs abs Absolute value true None
spark.rapids.sql.expression.Acos acos Inverse cosine true None
spark.rapids.sql.expression.Acosh acosh Inverse hyperbolic cosine true None
spark.rapids.sql.expression.Add + Addition true None
spark.rapids.sql.expression.Alias Gives a column a name true None
spark.rapids.sql.expression.And and Logical AND true None
spark.rapids.sql.expression.AnsiCast Convert a column of one type of data into another type true None
spark.rapids.sql.expression.Asin asin Inverse sine true None
spark.rapids.sql.expression.Asinh asinh Inverse hyperbolic sine true None
spark.rapids.sql.expression.AtLeastNNonNulls Checks if number of non null/Nan values is greater than a given value true None
spark.rapids.sql.expression.Atan atan Inverse tangent true None
spark.rapids.sql.expression.Atanh atanh Inverse hyperbolic tangent true None
spark.rapids.sql.expression.AttributeReference References an input column true None
spark.rapids.sql.expression.BitwiseAnd & Returns the bitwise AND of the operands true None
spark.rapids.sql.expression.BitwiseNot ~ Returns the bitwise NOT of the operands true None
spark.rapids.sql.expression.BitwiseOr | Returns the bitwise OR of the operands true None
spark.rapids.sql.expression.BitwiseXor ^ Returns the bitwise XOR of the operands true None
spark.rapids.sql.expression.CaseWhen when CASE WHEN expression true None
spark.rapids.sql.expression.Cast timestamp, tinyint, binary, float, smallint, string, decimal, double, boolean, cast, date, int, bigint Convert a column of one type of data into another type true None
spark.rapids.sql.expression.Cbrt cbrt Cube root true None
spark.rapids.sql.expression.Ceil ceiling, ceil Ceiling of a number true None
spark.rapids.sql.expression.Coalesce coalesce Returns the first non-null argument if exists. Otherwise, null true None
spark.rapids.sql.expression.Concat concat String concatenate NO separator true None
spark.rapids.sql.expression.Contains Contains true None
spark.rapids.sql.expression.Cos cos Cosine true None
spark.rapids.sql.expression.Cosh cosh Hyperbolic cosine true None
spark.rapids.sql.expression.Cot cot Cotangent true None
spark.rapids.sql.expression.CurrentRow$ Special boundary for a window frame, indicating stopping at the current row true None
spark.rapids.sql.expression.DateAdd date_add Returns the date that is num_days after start_date true None
spark.rapids.sql.expression.DateDiff datediff Returns the number of days from startDate to endDate true None
spark.rapids.sql.expression.DateSub date_sub Returns the date that is num_days before start_date true None
spark.rapids.sql.expression.DayOfMonth dayofmonth, day Returns the day of the month from a date or timestamp true None
spark.rapids.sql.expression.DayOfWeek dayofweek Returns the day of the week (1 = Sunday...7=Saturday) true None
spark.rapids.sql.expression.DayOfYear dayofyear Returns the day of the year from a date or timestamp true None
spark.rapids.sql.expression.Divide / Division true None
spark.rapids.sql.expression.EndsWith Ends with true None
spark.rapids.sql.expression.EqualNullSafe <=> Check if the values are equal including nulls <=> true None
spark.rapids.sql.expression.EqualTo =, == Check if the values are equal true None
spark.rapids.sql.expression.Exp exp Euler's number e raised to a power true None
spark.rapids.sql.expression.Expm1 expm1 Euler's number e raised to a power minus 1 true None
spark.rapids.sql.expression.Floor floor Floor of a number true None
spark.rapids.sql.expression.FromUnixTime from_unixtime Get the string from a unix timestamp true None
spark.rapids.sql.expression.GetArrayItem Gets the field at ordinal in the Array true None
spark.rapids.sql.expression.GreaterThan > > operator true None
spark.rapids.sql.expression.GreaterThanOrEqual >= >= operator true None
spark.rapids.sql.expression.Hour hour Returns the hour component of the string/timestamp true None
spark.rapids.sql.expression.If if IF expression true None
spark.rapids.sql.expression.In in IN operator true None
spark.rapids.sql.expression.InSet INSET operator true None
spark.rapids.sql.expression.InitCap initcap Returns str with the first letter of each word in uppercase. All other letters are in lowercase false This is not 100% compatible with the Spark version because in some cases unicode characters change byte width when changing the case. The GPU string conversion does not support these characters. For a full list of unsupported characters see rapidsai/cudf#3132 Spark also only sees the space character as a word deliminator, but this uses more white space characters.
spark.rapids.sql.expression.InputFileBlockLength input_file_block_length Returns the length of the block being read, or -1 if not available true None
spark.rapids.sql.expression.InputFileBlockStart input_file_block_start Returns the start offset of the block being read, or -1 if not available true None
spark.rapids.sql.expression.InputFileName input_file_name Returns the name of the file being read, or empty string if not available true None
spark.rapids.sql.expression.IntegralDivide div Division with a integer result true None
spark.rapids.sql.expression.IsNaN isnan Checks if a value is NaN true None
spark.rapids.sql.expression.IsNotNull isnotnull Checks if a value is not null true None
spark.rapids.sql.expression.IsNull isnull Checks if a value is null true None
spark.rapids.sql.expression.KnownFloatingPointNormalized Tag to prevent redundant normalization true None
spark.rapids.sql.expression.LastDay last_day Returns the last day of the month which the date belongs to true None
spark.rapids.sql.expression.Length length, character_length, char_length String character length true None
spark.rapids.sql.expression.LessThan < < operator true None
spark.rapids.sql.expression.LessThanOrEqual <= <= operator true None
spark.rapids.sql.expression.Like like Like true None
spark.rapids.sql.expression.Literal Holds a static value from the query true None
spark.rapids.sql.expression.Log ln Natural log true None
spark.rapids.sql.expression.Log10 log10 Log base 10 true None
spark.rapids.sql.expression.Log1p log1p Natural log 1 + expr true None
spark.rapids.sql.expression.Log2 log2 Log base 2 true None
spark.rapids.sql.expression.Logarithm log Log variable base true None
spark.rapids.sql.expression.Lower lower, lcase String lowercase operator false This is not 100% compatible with the Spark version because in some cases unicode characters change byte width when changing the case. The GPU string conversion does not support these characters. For a full list of unsupported characters see rapidsai/cudf#3132
spark.rapids.sql.expression.Minute minute Returns the minute component of the string/timestamp true None
spark.rapids.sql.expression.MonotonicallyIncreasingID monotonically_increasing_id Returns monotonically increasing 64-bit integers true None
spark.rapids.sql.expression.Month month Returns the month from a date or timestamp true None
spark.rapids.sql.expression.Multiply * Multiplication true None
spark.rapids.sql.expression.NaNvl nanvl Evaluates to left iff left is not NaN, right otherwise true None
spark.rapids.sql.expression.Not !, not Boolean not operator true None
spark.rapids.sql.expression.Or or Logical OR true None
spark.rapids.sql.expression.Pmod pmod Pmod true None
spark.rapids.sql.expression.Pow pow, power lhs ^ rhs true None
spark.rapids.sql.expression.PythonUDF UDF run in an external python process. Does not actually run on the GPU, but the transfer of data to/from it can be accelerated. true None
spark.rapids.sql.expression.Quarter quarter Returns the quarter of the year for date, in the range 1 to 4 true None
spark.rapids.sql.expression.Rand random, rand Generate a random column with i.i.d. uniformly distributed values in [0, 1) true None
spark.rapids.sql.expression.RegExpReplace regexp_replace RegExpReplace support for string literal input patterns true None
spark.rapids.sql.expression.Remainder %, mod Remainder or modulo true None
spark.rapids.sql.expression.Rint rint Rounds up a double value to the nearest double equal to an integer true None
spark.rapids.sql.expression.RowNumber row_number Window function that returns the index for the row within the aggregation window true None
spark.rapids.sql.expression.Second second Returns the second component of the string/timestamp true None
spark.rapids.sql.expression.ShiftLeft shiftleft Bitwise shift left (<<) true None
spark.rapids.sql.expression.ShiftRight shiftright Bitwise shift right (>>) true None
spark.rapids.sql.expression.ShiftRightUnsigned shiftrightunsigned Bitwise unsigned shift right (>>>) true None
spark.rapids.sql.expression.Signum sign, signum Returns -1.0, 0.0 or 1.0 as expr is negative, 0 or positive true None
spark.rapids.sql.expression.Sin sin Sine true None
spark.rapids.sql.expression.Sinh sinh Hyperbolic sine true None
spark.rapids.sql.expression.SortOrder Sort order true None
spark.rapids.sql.expression.SparkPartitionID spark_partition_id Returns the current partition id true None
spark.rapids.sql.expression.SpecifiedWindowFrame Specification of the width of the group (or "frame") of input rows around which a window function is evaluated true None
spark.rapids.sql.expression.Sqrt sqrt Square root true None
spark.rapids.sql.expression.StartsWith Starts with true None
spark.rapids.sql.expression.StringLPad lpad Pad a string on the left true None
spark.rapids.sql.expression.StringLocate position, locate Substring search operator true None
spark.rapids.sql.expression.StringRPad rpad Pad a string on the right true None
spark.rapids.sql.expression.StringReplace replace StringReplace operator true None
spark.rapids.sql.expression.StringSplit split Splits str around occurrences that match regex true None
spark.rapids.sql.expression.StringTrim trim StringTrim operator true None
spark.rapids.sql.expression.StringTrimLeft ltrim StringTrimLeft operator true None
spark.rapids.sql.expression.StringTrimRight rtrim StringTrimRight operator true None
spark.rapids.sql.expression.Substring substr, substring Substring operator true None
spark.rapids.sql.expression.SubstringIndex substring_index substring_index operator true None
spark.rapids.sql.expression.Subtract - Subtraction true None
spark.rapids.sql.expression.Tan tan Tangent true None
spark.rapids.sql.expression.Tanh tanh Hyperbolic tangent true None
spark.rapids.sql.expression.TimeAdd Adds interval to timestamp true None
spark.rapids.sql.expression.TimeSub Subtracts interval from timestamp true None
spark.rapids.sql.expression.ToDegrees degrees Converts radians to degrees true None
spark.rapids.sql.expression.ToRadians radians Converts degrees to radians true None
spark.rapids.sql.expression.ToUnixTimestamp to_unix_timestamp Returns the UNIX timestamp of the given time false This is not 100% compatible with the Spark version because Incorrectly formatted strings and bogus dates produce garbage data instead of null
spark.rapids.sql.expression.UnaryMinus negative Negate a numeric value true None
spark.rapids.sql.expression.UnaryPositive positive A numeric value with a + in front of it true None
spark.rapids.sql.expression.UnboundedFollowing$ Special boundary for a window frame, indicating all rows preceding the current row true None
spark.rapids.sql.expression.UnboundedPreceding$ Special boundary for a window frame, indicating all rows preceding the current row true None
spark.rapids.sql.expression.UnixTimestamp unix_timestamp Returns the UNIX timestamp of current or specified time false This is not 100% compatible with the Spark version because Incorrectly formatted strings and bogus dates produce garbage data instead of null
spark.rapids.sql.expression.Upper upper, ucase String uppercase operator false This is not 100% compatible with the Spark version because in some cases unicode characters change byte width when changing the case. The GPU string conversion does not support these characters. For a full list of unsupported characters see rapidsai/cudf#3132
spark.rapids.sql.expression.WeekDay weekday Returns the day of the week (0 = Monday...6=Sunday) true None
spark.rapids.sql.expression.WindowExpression Calculates a return value for every input row of a table based on a group (or "window") of rows true None
spark.rapids.sql.expression.WindowSpecDefinition Specification of a window function, indicating the partitioning-expression, the row ordering, and the width of the window true None
spark.rapids.sql.expression.Year year Returns the year from a date or timestamp true None
spark.rapids.sql.expression.AggregateExpression Aggregate expression true None
spark.rapids.sql.expression.Average avg, mean Average aggregate operator true None
spark.rapids.sql.expression.Count count Count aggregate operator true None
spark.rapids.sql.expression.First first_value, first first aggregate operator true None
spark.rapids.sql.expression.Last last, last_value last aggregate operator true None
spark.rapids.sql.expression.Max max Max aggregate operator true None
spark.rapids.sql.expression.Min min Min aggregate operator true None
spark.rapids.sql.expression.Sum sum Sum aggregate operator true None
spark.rapids.sql.expression.NormalizeNaNAndZero Normalize NaN and zero true None

Execution

Name Description Default Value Notes
spark.rapids.sql.exec.CoalesceExec The backend for the dataframe coalesce method true None
spark.rapids.sql.exec.CollectLimitExec Reduce to single partition and apply limit true None
spark.rapids.sql.exec.ExpandExec The backend for the expand operator true None
spark.rapids.sql.exec.FileSourceScanExec Reading data from files, often from Hive tables true None
spark.rapids.sql.exec.FilterExec The backend for most filter statements true None
spark.rapids.sql.exec.GenerateExec The backend for operations that generate more output rows than input rows like explode true None
spark.rapids.sql.exec.GlobalLimitExec Limiting of results across partitions true None
spark.rapids.sql.exec.LocalLimitExec Per-partition limiting of results true None
spark.rapids.sql.exec.ProjectExec The backend for most select, withColumn and dropColumn statements true None
spark.rapids.sql.exec.RangeExec The backend for range operator true None
spark.rapids.sql.exec.SortExec The backend for the sort operator true None
spark.rapids.sql.exec.UnionExec The backend for the union operator true None
spark.rapids.sql.exec.CustomShuffleReaderExec A wrapper of shuffle query stage true None
spark.rapids.sql.exec.HashAggregateExec The backend for hash based aggregations true None
spark.rapids.sql.exec.SortAggregateExec The backend for sort based aggregations true None
spark.rapids.sql.exec.DataWritingCommandExec Writing data true None
spark.rapids.sql.exec.BatchScanExec The backend for most file input true None
spark.rapids.sql.exec.BroadcastExchangeExec The backend for broadcast exchange of data true None
spark.rapids.sql.exec.ShuffleExchangeExec The backend for most data being exchanged between processes true None
spark.rapids.sql.exec.BroadcastHashJoinExec Implementation of join using broadcast data true None
spark.rapids.sql.exec.BroadcastNestedLoopJoinExec Implementation of join using brute force false This is disabled by default because large joins can cause out of memory errors
spark.rapids.sql.exec.CartesianProductExec Implementation of join using brute force false This is disabled by default because large joins can cause out of memory errors
spark.rapids.sql.exec.ShuffledHashJoinExec Implementation of join using hashed shuffled data true None
spark.rapids.sql.exec.SortMergeJoinExec Sort merge join, replacing with shuffled hash join true None
spark.rapids.sql.exec.AggregateInPandasExec The backend for Grouped Aggregation Pandas UDF, it runs on CPU itself now but supports running the Python UDFs code on GPU when calling cuDF APIs in the UDF false This is disabled by default because Performance is not ideal now
spark.rapids.sql.exec.ArrowEvalPythonExec The backend of the Scalar Pandas UDFs, it supports running the Python UDFs code on GPU when calling cuDF APIs in the UDF, also accelerates the data transfer between the Java process and Python process false This is disabled by default because Performance is not ideal for UDFs that take a long time
spark.rapids.sql.exec.FlatMapCoGroupsInPandasExec The backend for CoGrouped Aggregation Pandas UDF, it runs on CPU itself now but supports running the Python UDFs code on GPU when calling cuDF APIs in the UDF false This is disabled by default because Performance is not ideal now
spark.rapids.sql.exec.FlatMapGroupsInPandasExec The backend for Grouped Map Pandas UDF, it runs on CPU itself now but supports running the Python UDFs code on GPU when calling cuDF APIs in the UDF false This is disabled by default because Performance is not ideal now
spark.rapids.sql.exec.MapInPandasExec The backend for Map Pandas Iterator UDF, it runs on CPU itself now but supports running the Python UDFs code on GPU when calling cuDF APIs in the UDF false This is disabled by default because Performance is not ideal now
spark.rapids.sql.exec.WindowInPandasExec The backend for Pandas UDF with window functions, it runs on CPU itself now but supports running the Python UDFs code on GPU when calling cuDF APIs in the UDF false This is disabled by default because Performance is not ideal now
spark.rapids.sql.exec.WindowExec Window-operator backend true None

Scans

Name Description Default Value Notes
spark.rapids.sql.input.CSVScan CSV parsing true None
spark.rapids.sql.input.OrcScan ORC parsing true None
spark.rapids.sql.input.ParquetScan Parquet parsing true None

Partitioning

Name Description Default Value Notes
spark.rapids.sql.partitioning.HashPartitioning Hash based partitioning true None
spark.rapids.sql.partitioning.RangePartitioning Range partitioning true None
spark.rapids.sql.partitioning.RoundRobinPartitioning Round robin partitioning true None
spark.rapids.sql.partitioning.SinglePartition$ Single partitioning true None

JIT Kernel Cache Path

CUDF can compile GPU kernels at runtime using a just-in-time (JIT) compiler. The resulting kernels are cached on the filesystem. The default location for this cache is under the .cudf directory in the user's home directory. When running in an environment where the user's home directory cannot be written, such as running in a container environment on a cluster, the JIT cache path will need to be specified explicitly with the LIBCUDF_KERNEL_CACHE_PATH environment variable. The specified kernel cache path should be specific to the user to avoid conflicts with others running on the same host. For example, the following would specify the path to a user-specific location under /tmp:

--conf spark.executorEnv.LIBCUDF_KERNEL_CACHE_PATH="/tmp/cudf-$USER"