Skip to content
Closed
Show file tree
Hide file tree
Changes from all commits
Commits
File filter

Filter by extension

Filter by extension

Conversations
Failed to load comments.
Loading
Jump to
Jump to file
Failed to load files.
Loading
Diff view
Diff view
Original file line number Diff line number Diff line change
Expand Up @@ -25,6 +25,7 @@
import org.apache.avro.file.CodecFactory;
import org.apache.avro.file.DataFileWriter;
import org.apache.avro.generic.GenericData;
import org.apache.avro.generic.GenericDatumWriter;
import org.apache.avro.generic.GenericRecord;
import org.apache.avro.mapred.AvroKey;
import org.apache.avro.mapreduce.AvroKeyOutputFormat;
Expand Down Expand Up @@ -53,7 +54,7 @@ protected RecordWriter<AvroKey<GenericRecord>, NullWritable> create(
CodecFactory compressionCodec,
OutputStream outputStream,
int syncInterval) throws IOException {
return new SparkAvroKeyRecordWriter(
return new SparkAvroKeyRecordWriter<>(
writerSchema, dataModel, compressionCodec, outputStream, syncInterval, metadata);
}
}
Expand All @@ -72,7 +73,7 @@ class SparkAvroKeyRecordWriter<T> extends RecordWriter<AvroKey<T>, NullWritable>
OutputStream outputStream,
int syncInterval,
Map<String, String> metadata) throws IOException {
this.mAvroFileWriter = new DataFileWriter(dataModel.createDatumWriter(writerSchema));
this.mAvroFileWriter = new DataFileWriter<>(new GenericDatumWriter<>(writerSchema, dataModel));
Copy link
Contributor Author

@LuciferYang LuciferYang Oct 11, 2022

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

for (Map.Entry<String, String> entry : metadata.entrySet()) {
this.mAvroFileWriter.setMeta(entry.getKey(), entry.getValue());
}
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -1075,7 +1075,7 @@ abstract class AvroSuite
.save(s"$tempDir/${UUID.randomUUID()}")
}.getMessage
assert(message.contains("Caused by: java.lang.NullPointerException: "))
assert(message.contains("null in string in field Name"))
assert(message.contains("null value for (non-nullable) string at test_schema.Name"))
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

The error message change from

Job aborted due to stage failure: Task 1 in stage 1.0 failed 1 times, most recent failure: Lost task 1.0 in stage 1.0 (TID 3) (localhost executor driver): org.apache.spark.SparkException: Task failed while writing rows.
	at org.apache.spark.sql.errors.QueryExecutionErrors$.taskFailedWhileWritingRowsError(QueryExecutionErrors.scala:723)
	at org.apache.spark.sql.execution.datasources.FileFormatWriter$.executeTask(FileFormatWriter.scala:310)
	at org.apache.spark.sql.execution.datasources.FileFormatWriter$.$anonfun$write$11(FileFormatWriter.scala:217)
	at org.apache.spark.scheduler.ResultTask.runTask(ResultTask.scala:92)
	at org.apache.spark.TaskContext.runTaskWithListeners(TaskContext.scala:161)
	at org.apache.spark.scheduler.Task.run(Task.scala:139)
	at org.apache.spark.executor.Executor$TaskRunner.$anonfun$run$3(Executor.scala:554)
	at org.apache.spark.util.Utils$.tryWithSafeFinally(Utils.scala:1491)
	at org.apache.spark.executor.Executor$TaskRunner.run(Executor.scala:557)
	at java.base/java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1136)
	at java.base/java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:635)
	at java.base/java.lang.Thread.run(Thread.java:833)
Caused by: org.apache.avro.file.DataFileWriter$AppendWriteException: java.lang.NullPointerException: null in test_schema
	at org.apache.avro.file.DataFileWriter.append(DataFileWriter.java:317)
	at org.apache.spark.sql.avro.SparkAvroKeyRecordWriter.write(SparkAvroKeyOutputFormat.java:87)
	at org.apache.spark.sql.avro.SparkAvroKeyRecordWriter.write(SparkAvroKeyOutputFormat.java:64)
	at org.apache.spark.sql.avro.AvroOutputWriter.write(AvroOutputWriter.scala:86)
	at org.apache.spark.sql.execution.datasources.SingleDirectoryDataWriter.write(FileFormatDataWriter.scala:175)
	at org.apache.spark.sql.execution.datasources.FileFormatDataWriter.writeWithMetrics(FileFormatDataWriter.scala:85)
	at org.apache.spark.sql.execution.datasources.FileFormatDataWriter.writeWithIterator(FileFormatDataWriter.scala:92)
	at org.apache.spark.sql.execution.datasources.FileFormatWriter$.$anonfun$executeTask$1(FileFormatWriter.scala:293)
	at org.apache.spark.util.Utils$.tryWithSafeFinallyAndFailureCallbacks(Utils.scala:1525)
	at org.apache.spark.sql.execution.datasources.FileFormatWriter$.executeTask(FileFormatWriter.scala:300)
	... 10 more
Caused by: java.lang.NullPointerException: null in test_schema
	at org.apache.avro.generic.GenericDatumWriter.npe(GenericDatumWriter.java:208)
	at org.apache.avro.reflect.ReflectDatumWriter.write(ReflectDatumWriter.java:160)
	at org.apache.avro.generic.GenericDatumWriter.write(GenericDatumWriter.java:82)
	at org.apache.avro.file.DataFileWriter.append(DataFileWriter.java:314)
	... 19 more
Caused by: java.lang.NullPointerException: null in string in field Name
	at org.apache.avro.generic.GenericDatumWriter.npe(GenericDatumWriter.java:208)
	at org.apache.avro.generic.GenericDatumWriter.writeField(GenericDatumWriter.java:254)
	at org.apache.avro.specific.SpecificDatumWriter.writeField(SpecificDatumWriter.java:117)
	at org.apache.avro.reflect.ReflectDatumWriter.writeField(ReflectDatumWriter.java:184)
	at org.apache.avro.generic.GenericDatumWriter.writeRecord(GenericDatumWriter.java:234)
	at org.apache.avro.specific.SpecificDatumWriter.writeRecord(SpecificDatumWriter.java:92)
	at org.apache.avro.generic.GenericDatumWriter.writeWithoutConversion(GenericDatumWriter.java:145)
	at org.apache.avro.generic.GenericDatumWriter.write(GenericDatumWriter.java:95)
	at org.apache.avro.reflect.ReflectDatumWriter.write(ReflectDatumWriter.java:158)
	... 21 more
Caused by: java.lang.NullPointerException: Cannot invoke "Object.getClass()" because "datum" is null
	at org.apache.avro.specific.SpecificDatumWriter.writeString(SpecificDatumWriter.java:73)
	at org.apache.avro.generic.GenericDatumWriter.writeWithoutConversion(GenericDatumWriter.java:165)
	at org.apache.avro.generic.GenericDatumWriter.write(GenericDatumWriter.java:95)
	at org.apache.avro.reflect.ReflectDatumWriter.write(ReflectDatumWriter.java:158)
	at org.apache.avro.generic.GenericDatumWriter.writeField(GenericDatumWriter.java:245)
	... 28 more

to

"Job aborted due to stage failure: Task 1 in stage 148.0 failed 1 times, most recent failure: Lost task 1.0 in stage 148.0 (TID 252) (localhost executor driver): org.apache.spark.SparkException: Task failed while writing rows.
  at org.apache.spark.sql.errors.QueryExecutionErrors$.taskFailedWhileWritingRowsError(QueryExecutionErrors.scala:723)
  at org.apache.spark.sql.execution.datasources.FileFormatWriter$.executeTask(FileFormatWriter.scala:310)
  at org.apache.spark.sql.execution.datasources.FileFormatWriter$.$anonfun$write$11(FileFormatWriter.scala:217)
  at org.apache.spark.scheduler.ResultTask.runTask(ResultTask.scala:92)
  at org.apache.spark.TaskContext.runTaskWithListeners(TaskContext.scala:161)
  at org.apache.spark.scheduler.Task.run(Task.scala:139)
  at org.apache.spark.executor.Executor$TaskRunner.$anonfun$run$3(Executor.scala:554)
  at org.apache.spark.util.Utils$.tryWithSafeFinally(Utils.scala:1491)
  at org.apache.spark.executor.Executor$TaskRunner.run(Executor.scala:557)
  at java.base/java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1136)
  at java.base/java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:635)
  at java.base/java.lang.Thread.run(Thread.java:833)
Caused by: org.apache.avro.file.DataFileWriter$AppendWriteException: java.lang.NullPointerException: null value for (non-nullable) string at test_schema.Name
  at org.apache.avro.file.DataFileWriter.append(DataFileWriter.java:317)
  at org.apache.spark.sql.avro.SparkAvroKeyRecordWriter.write(SparkAvroKeyOutputFormat.java:87)
  at org.apache.spark.sql.avro.SparkAvroKeyRecordWriter.write(SparkAvroKeyOutputFormat.java:64)
  at org.apache.spark.sql.avro.AvroOutputWriter.write(AvroOutputWriter.scala:86)
  at org.apache.spark.sql.execution.datasources.SingleDirectoryDataWriter.write(FileFormatDataWriter.scala:175)
  at org.apache.spark.sql.execution.datasources.FileFormatDataWriter.writeWithMetrics(FileFormatDataWriter.scala:85)
  at org.apache.spark.sql.execution.datasources.FileFormatDataWriter.writeWithIterator(FileFormatDataWriter.scala:92)
  at org.apache.spark.sql.execution.datasources.FileFormatWriter$.$anonfun$executeTask$1(FileFormatWriter.scala:293)
  at org.apache.spark.util.Utils$.tryWithSafeFinallyAndFailureCallbacks(Utils.scala:1525)
  at org.apache.spark.sql.execution.datasources.FileFormatWriter$.executeTask(FileFormatWriter.scala:300)
  ... 10 more
Caused by: java.lang.NullPointerException: null value for (non-nullable) string at test_schema.Name
  at org.apache.avro.path.TracingNullPointException.summarize(TracingNullPointException.java:88)
  at org.apache.avro.path.TracingNullPointException.summarize(TracingNullPointException.java:30)
  at org.apache.avro.generic.GenericDatumWriter.write(GenericDatumWriter.java:84)
  at org.apache.avro.file.DataFileWriter.append(DataFileWriter.java:314)
  ... 19 more
Caused by: java.lang.NullPointerException: Cannot invoke "java.lang.CharSequence.toString()" because "charSequence" is null
  at org.apache.avro.io.Encoder.writeString(Encoder.java:130)
  at org.apache.avro.generic.GenericDatumWriter.writeString(GenericDatumWriter.java:392)
  at org.apache.avro.generic.GenericDatumWriter.writeString(GenericDatumWriter.java:384)
  at org.apache.avro.generic.GenericDatumWriter.writeWithoutConversion(GenericDatumWriter.java:165)
  at org.apache.avro.generic.GenericDatumWriter.write(GenericDatumWriter.java:95)
  at org.apache.avro.generic.GenericDatumWriter.writeField(GenericDatumWriter.java:245)
  at org.apache.avro.generic.GenericDatumWriter.writeRecord(GenericDatumWriter.java:234)
  at org.apache.avro.generic.GenericDatumWriter.writeWithoutConversion(GenericDatumWriter.java:145)
  at org.apache.avro.generic.GenericDatumWriter.write(GenericDatumWriter.java:95)
  at org.apache.avro.generic.GenericDatumWriter.write(GenericDatumWriter.java:82)
  ... 20 more

due to change of SparkAvroKeyOutputFormat, the exception type not changed, but the error message has changed

}
}

Expand Down
2 changes: 1 addition & 1 deletion core/src/main/java/org/apache/spark/SparkThrowable.java
Original file line number Diff line number Diff line change
Expand Up @@ -51,7 +51,7 @@ default boolean isInternalError() {
}

default Map<String, String> getMessageParameters() {
return new HashMap();
return new HashMap<>();
}

default QueryContext[] getQueryContext() { return new QueryContext[0]; }
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -45,6 +45,7 @@
@Experimental
public interface SupportsAtomicPartitionManagement extends SupportsPartitionManagement {

@SuppressWarnings("unchecked")
@Override
default void createPartition(
Copy link
Contributor Author

Choose a reason for hiding this comment

The reason will be displayed to describe this comment to others. Learn more.

Can't new new Map<String, String>[]{properties}, just suppress it

InternalRow ident,
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -47,7 +47,7 @@ public class V2ExpressionSQLBuilder {

public String build(Expression expr) {
if (expr instanceof Literal) {
return visitLiteral((Literal) expr);
return visitLiteral((Literal<?>) expr);
} else if (expr instanceof NamedReference) {
return visitNamedReference((NamedReference) expr);
} else if (expr instanceof Cast) {
Expand Down Expand Up @@ -213,7 +213,7 @@ public String build(Expression expr) {
}
}

protected String visitLiteral(Literal literal) {
protected String visitLiteral(Literal<?> literal) {
return literal.toString();
}

Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -19,6 +19,7 @@

import java.util.ArrayList;
import java.util.Collections;
import java.util.List;
import java.util.Random;


Expand Down Expand Up @@ -53,20 +54,20 @@ public class NumericHistogram {
*
* @since 3.3.0
*/
public static class Coord implements Comparable {
public static class Coord implements Comparable<Coord> {
public double x;
public double y;

@Override
public int compareTo(Object other) {
return Double.compare(x, ((Coord) other).x);
public int compareTo(Coord other) {
return Double.compare(x, other.x);
}
}

// Class variables
private int nbins;
private int nusedbins;
private ArrayList<Coord> bins;
private List<Coord> bins;
private Random prng;

/**
Expand Down Expand Up @@ -146,7 +147,7 @@ public void addBin(double x, double y, int b) {
*/
public void allocate(int num_bins) {
nbins = num_bins;
bins = new ArrayList<Coord>();
bins = new ArrayList<>();
nusedbins = 0;
}

Expand All @@ -163,7 +164,7 @@ public void merge(NumericHistogram other) {
// by deserializing the ArrayList of (x,y) pairs into an array of Coord objects
nbins = other.nbins;
nusedbins = other.nusedbins;
bins = new ArrayList<Coord>(nusedbins);
bins = new ArrayList<>(nusedbins);
for (int i = 0; i < other.nusedbins; i += 1) {
Coord bin = new Coord();
bin.x = other.getBin(i).x;
Expand All @@ -174,7 +175,7 @@ public void merge(NumericHistogram other) {
// The aggregation buffer already contains a partial histogram. Therefore, we need
// to merge histograms using Algorithm #2 from the Ben-Haim and Tom-Tov paper.

ArrayList<Coord> tmp_bins = new ArrayList<Coord>(nusedbins + other.nusedbins);
List<Coord> tmp_bins = new ArrayList<>(nusedbins + other.nusedbins);
// Copy all the histogram bins from us and 'other' into an overstuffed histogram
for (int i = 0; i < nusedbins; i++) {
Coord bin = new Coord();
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -17,6 +17,7 @@

package org.apache.hive.service.cli.operation;
import java.io.CharArrayWriter;
import java.io.Serializable;
import java.util.Map;
import java.util.regex.Pattern;

Expand Down Expand Up @@ -265,7 +266,7 @@ private static StringLayout initLayout(OperationLog.LoggingLevel loggingMode) {
Map<String, Appender> appenders = root.getAppenders();
for (Appender ap : appenders.values()) {
if (ap.getClass().equals(ConsoleAppender.class)) {
Layout l = ap.getLayout();
Layout<? extends Serializable> l = ap.getLayout();
if (l instanceof StringLayout) {
layout = (StringLayout) l;
break;
Expand Down
Original file line number Diff line number Diff line change
Expand Up @@ -39,7 +39,7 @@
import org.apache.hive.service.cli.session.HiveSession;
import org.apache.hive.service.rpc.thrift.TRowSet;
import org.apache.hive.service.rpc.thrift.TTableSchema;
import org.apache.logging.log4j.core.appender.AbstractWriterAppender;
import org.apache.logging.log4j.core.Appender;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;

Expand Down Expand Up @@ -82,7 +82,7 @@ public synchronized void stop() {

private void initOperationLogCapture(String loggingMode) {
// Register another Appender (with the same layout) that talks to us.
AbstractWriterAppender ap = new LogDivertAppender(this, OperationLog.getLoggingLevel(loggingMode));
Appender ap = new LogDivertAppender(this, OperationLog.getLoggingLevel(loggingMode));
((org.apache.logging.log4j.core.Logger)org.apache.logging.log4j.LogManager.getRootLogger()).addAppender(ap);
ap.start();
}
Expand Down