Class RelationalGroupedDataset
DataFrame, created by groupBy,
 cube or rollup (and also pivot).
 
 The main method is the agg function, which has multiple variants. This class also contains
 some first-order statistics such as mean, sum for convenience.
 
- Since:
- 2.0.0
- Note:
- This class was named GroupedDatain Spark 1.x.
- 
Constructor SummaryConstructors
- 
Method SummaryModifier and TypeMethodDescription(Java-specific) Compute aggregates by specifying a map from column name to aggregate methods.Compute aggregates by specifying a series of aggregate columns.Compute aggregates by specifying a series of aggregate columns.(Scala-specific) Compute aggregates by specifying a map from column name to aggregate methods.agg(scala.Tuple2<String, String> aggExpr, scala.collection.immutable.Seq<scala.Tuple2<String, String>> aggExprs) (Scala-specific) Compute aggregates by specifying the column names and aggregate methods.abstract <K,T> KeyValueGroupedDataset<K, T> Returns aKeyValueGroupedDatasetwhere the data is grouped by the grouping expressions of currentRelationalGroupedDataset.Compute the mean value for each numeric columns for each group.Compute the mean value for each numeric columns for each group.count()Count the number of rows for each group.Compute the max value for each numeric columns for each group.Compute the max value for each numeric columns for each group.Compute the average value for each numeric columns for each group.Compute the average value for each numeric columns for each group.Compute the min value for each numeric column for each group.Compute the min value for each numeric column for each group.Pivots a column of the currentDataFrameand performs the specified aggregation.(Java-specific) Pivots a column of the currentDataFrameand performs the specified aggregation.Pivots a column of the currentDataFrameand performs the specified aggregation.abstract RelationalGroupedDatasetPivots a column of the currentDataFrameand performs the specified aggregation.(Java-specific) Pivots a column of the currentDataFrameand performs the specified aggregation.abstract RelationalGroupedDatasetPivots a column of the currentDataFrameand performs the specified aggregation.Compute the sum for each numeric columns for each group.Compute the sum for each numeric columns for each group.
- 
Constructor Details- 
RelationalGroupedDatasetpublic RelationalGroupedDataset()
 
- 
- 
Method Details- 
aggCompute aggregates by specifying a series of aggregate columns. Note that this function by default retains the grouping columns in its output. To not retain grouping columns, setspark.sql.retainGroupColumnsto false.The available aggregate methods are defined in functions.// Selects the age of the oldest employee and the aggregate expense for each department // Scala: import org.apache.spark.sql.functions._ df.groupBy("department").agg(max("age"), sum("expense")) // Java: import static org.apache.spark.sql.functions.*; df.groupBy("department").agg(max("age"), sum("expense"));Note that before Spark 1.4, the default behavior is to NOT retain grouping columns. To change to that behavior, set config variable spark.sql.retainGroupColumnstofalse.// Scala, 1.3.x: df.groupBy("department").agg($"department", max("age"), sum("expense")) // Java, 1.3.x: df.groupBy("department").agg(col("department"), max("age"), sum("expense"));- Parameters:
- expr- (undocumented)
- exprs- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
aggpublic Dataset<Row> agg(scala.Tuple2<String, String> aggExpr, scala.collection.immutable.Seq<scala.Tuple2<String, String>> aggExprs) (Scala-specific) Compute aggregates by specifying the column names and aggregate methods. The resultingDataFramewill also contain the grouping columns.The available aggregate methods are avg,max,min,sum,count.// Selects the age of the oldest employee and the aggregate expense for each department df.groupBy("department").agg( "age" -> "max", "expense" -> "sum" )- Parameters:
- aggExpr- (undocumented)
- aggExprs- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
agg(Scala-specific) Compute aggregates by specifying a map from column name to aggregate methods. The resultingDataFramewill also contain the grouping columns.The available aggregate methods are avg,max,min,sum,count.// Selects the age of the oldest employee and the aggregate expense for each department df.groupBy("department").agg(Map( "age" -> "max", "expense" -> "sum" ))- Parameters:
- exprs- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
agg(Java-specific) Compute aggregates by specifying a map from column name to aggregate methods. The resultingDataFramewill also contain the grouping columns.The available aggregate methods are avg,max,min,sum,count.// Selects the age of the oldest employee and the aggregate expense for each department import com.google.common.collect.ImmutableMap; df.groupBy("department").agg(ImmutableMap.of("age", "max", "expense", "sum"));- Parameters:
- exprs- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
aggCompute aggregates by specifying a series of aggregate columns. Note that this function by default retains the grouping columns in its output. To not retain grouping columns, setspark.sql.retainGroupColumnsto false.The available aggregate methods are defined in functions.// Selects the age of the oldest employee and the aggregate expense for each department // Scala: import org.apache.spark.sql.functions._ df.groupBy("department").agg(max("age"), sum("expense")) // Java: import static org.apache.spark.sql.functions.*; df.groupBy("department").agg(max("age"), sum("expense"));Note that before Spark 1.4, the default behavior is to NOT retain grouping columns. To change to that behavior, set config variable spark.sql.retainGroupColumnstofalse.// Scala, 1.3.x: df.groupBy("department").agg($"department", max("age"), sum("expense")) // Java, 1.3.x: df.groupBy("department").agg(col("department"), max("age"), sum("expense"));- Parameters:
- expr- (undocumented)
- exprs- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
asReturns aKeyValueGroupedDatasetwhere the data is grouped by the grouping expressions of currentRelationalGroupedDataset.- Parameters:
- evidence$1- (undocumented)
- evidence$2- (undocumented)
- Returns:
- (undocumented)
- Since:
- 3.0.0
 
- 
avgCompute the mean value for each numeric columns for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the mean values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
avgCompute the mean value for each numeric columns for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the mean values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
countCount the number of rows for each group. The resultingDataFramewill also contain the grouping columns.- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
maxCompute the max value for each numeric columns for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the max values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
maxCompute the max value for each numeric columns for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the max values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
meanCompute the average value for each numeric columns for each group. This is an alias foravg. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the average values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
meanCompute the average value for each numeric columns for each group. This is an alias foravg. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the average values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
minCompute the min value for each numeric column for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the min values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
minCompute the min value for each numeric column for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the min values for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
pivotPivots a column of the currentDataFrameand performs the specified aggregation.Spark will eagerly compute the distinct values in pivotColumnso it can determine the resulting schema of the transformation. To avoid any eager computations, provide an explicit list of values viapivot(pivotColumn: String, values: Seq[Any]).// Compute the sum of earnings for each year by course with each course as a separate column df.groupBy("year").pivot("course").sum("earnings")- Parameters:
- pivotColumn- Name of the column to pivot.
- Returns:
- (undocumented)
- Since:
- 1.6.0
- See Also:
- 
- org.apache.spark.sql.Dataset.unpivotfor the reverse operation, except for the aggregation.
 
 
- 
pivotpublic RelationalGroupedDataset pivot(String pivotColumn, scala.collection.immutable.Seq<Object> values) Pivots a column of the currentDataFrameand performs the specified aggregation. There are two versions of pivot function: one that requires the caller to specify the list of distinct values to pivot on, and one that does not. The latter is more concise but less efficient, because Spark needs to first compute the list of distinct values internally.// Compute the sum of earnings for each year by course with each course as a separate column df.groupBy("year").pivot("course", Seq("dotNET", "Java")).sum("earnings") // Or without specifying column values (less efficient) df.groupBy("year").pivot("course").sum("earnings")From Spark 3.0.0, values can be literal columns, for instance, struct. For pivoting by multiple columns, use the structfunction to combine the columns and values:df.groupBy("year") .pivot("trainingCourse", Seq(struct(lit("java"), lit("Experts")))) .agg(sum($"earnings"))- Parameters:
- pivotColumn- Name of the column to pivot.
- values- List of values that will be translated to columns in the output DataFrame.
- Returns:
- (undocumented)
- Since:
- 1.6.0
- See Also:
- 
- org.apache.spark.sql.Dataset.unpivotfor the reverse operation, except for the aggregation.
 
 
- 
pivot(Java-specific) Pivots a column of the currentDataFrameand performs the specified aggregation.There are two versions of pivot function: one that requires the caller to specify the list of distinct values to pivot on, and one that does not. The latter is more concise but less efficient, because Spark needs to first compute the list of distinct values internally. // Compute the sum of earnings for each year by course with each course as a separate column df.groupBy("year").pivot("course", Arrays.<Object>asList("dotNET", "Java")).sum("earnings"); // Or without specifying column values (less efficient) df.groupBy("year").pivot("course").sum("earnings");- Parameters:
- pivotColumn- Name of the column to pivot.
- values- List of values that will be translated to columns in the output DataFrame.
- Returns:
- (undocumented)
- Since:
- 1.6.0
- See Also:
- 
- org.apache.spark.sql.Dataset.unpivotfor the reverse operation, except for the aggregation.
 
 
- 
pivot(Java-specific) Pivots a column of the currentDataFrameand performs the specified aggregation. This is an overloaded version of thepivotmethod withpivotColumnof theStringtype.- Parameters:
- pivotColumn- the column to pivot.
- values- List of values that will be translated to columns in the output DataFrame.
- Returns:
- (undocumented)
- Since:
- 2.4.0
- See Also:
- 
- org.apache.spark.sql.Dataset.unpivotfor the reverse operation, except for the aggregation.
 
 
- 
pivotPivots a column of the currentDataFrameand performs the specified aggregation.Spark will eagerly compute the distinct values in pivotColumnso it can determine the resulting schema of the transformation. To avoid any eager computations, provide an explicit list of values viapivot(pivotColumn: Column, values: Seq[Any]).// Compute the sum of earnings for each year by course with each course as a separate column df.groupBy($"year").pivot($"course").sum($"earnings");- Parameters:
- pivotColumn- he column to pivot.
- Returns:
- (undocumented)
- Since:
- 2.4.0
- See Also:
- 
- org.apache.spark.sql.Dataset.unpivotfor the reverse operation, except for the aggregation.
 
 
- 
pivotpublic abstract RelationalGroupedDataset pivot(Column pivotColumn, scala.collection.immutable.Seq<Object> values) Pivots a column of the currentDataFrameand performs the specified aggregation. This is an overloaded version of thepivotmethod withpivotColumnof theStringtype.// Compute the sum of earnings for each year by course with each course as a separate column df.groupBy($"year").pivot($"course", Seq("dotNET", "Java")).sum($"earnings")- Parameters:
- pivotColumn- the column to pivot.
- values- List of values that will be translated to columns in the output DataFrame.
- Returns:
- (undocumented)
- Since:
- 2.4.0
- See Also:
- 
- org.apache.spark.sql.Dataset.unpivotfor the reverse operation, except for the aggregation.
 
 
- 
sumCompute the sum for each numeric columns for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the sum for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
- 
sumCompute the sum for each numeric columns for each group. The resultingDataFramewill also contain the grouping columns. When specified columns are given, only compute the sum for them.- Parameters:
- colNames- (undocumented)
- Returns:
- (undocumented)
- Since:
- 1.3.0
 
 
-