Skip to content

Commit 92d931b

Browse files
Build: Fix errorprone warnings (apache#13217)
1 parent 7443e54 commit 92d931b

File tree

6 files changed

+94
-144
lines changed

6 files changed

+94
-144
lines changed

api/src/main/java/org/apache/iceberg/transforms/Timestamps.java

Lines changed: 65 additions & 123 deletions
Original file line numberDiff line numberDiff line change
@@ -32,24 +32,78 @@
3232
import org.apache.iceberg.util.SerializableFunction;
3333

3434
enum Timestamps implements Transform<Long, Integer> {
35-
MICROS_TO_YEAR(ChronoUnit.YEARS, "year", MicrosToYears.INSTANCE),
36-
MICROS_TO_MONTH(ChronoUnit.MONTHS, "month", MicrosToMonths.INSTANCE),
37-
MICROS_TO_DAY(ChronoUnit.DAYS, "day", MicrosToDays.INSTANCE),
38-
MICROS_TO_HOUR(ChronoUnit.HOURS, "hour", MicrosToHours.INSTANCE),
35+
MICROS_TO_YEAR(ChronoUnit.YEARS, "year", TimestampUnit.MICROS),
36+
MICROS_TO_MONTH(ChronoUnit.MONTHS, "month", TimestampUnit.MICROS),
37+
MICROS_TO_DAY(ChronoUnit.DAYS, "day", TimestampUnit.MICROS),
38+
MICROS_TO_HOUR(ChronoUnit.HOURS, "hour", TimestampUnit.MICROS),
3939

40-
NANOS_TO_YEAR(ChronoUnit.YEARS, "year", NanosToYears.INSTANCE),
41-
NANOS_TO_MONTH(ChronoUnit.MONTHS, "month", NanosToMonths.INSTANCE),
42-
NANOS_TO_DAY(ChronoUnit.DAYS, "day", NanosToDays.INSTANCE),
43-
NANOS_TO_HOUR(ChronoUnit.HOURS, "hour", NanosToHours.INSTANCE);
40+
NANOS_TO_YEAR(ChronoUnit.YEARS, "year", TimestampUnit.NANOS),
41+
NANOS_TO_MONTH(ChronoUnit.MONTHS, "month", TimestampUnit.NANOS),
42+
NANOS_TO_DAY(ChronoUnit.DAYS, "day", TimestampUnit.NANOS),
43+
NANOS_TO_HOUR(ChronoUnit.HOURS, "hour", TimestampUnit.NANOS);
44+
45+
enum TimestampUnit {
46+
MICROS,
47+
NANOS
48+
}
49+
50+
@Immutable
51+
static class Apply implements SerializableFunction<Long, Integer> {
52+
private final ChronoUnit granularity;
53+
private final TimestampUnit timestampUnit;
54+
55+
Apply(ChronoUnit granularity, TimestampUnit timestampUnit) {
56+
this.granularity = granularity;
57+
this.timestampUnit = timestampUnit;
58+
}
59+
60+
@Override
61+
public Integer apply(Long timestamp) {
62+
if (timestamp == null) {
63+
return null;
64+
}
65+
66+
switch (timestampUnit) {
67+
case MICROS:
68+
switch (granularity) {
69+
case YEARS:
70+
return DateTimeUtil.microsToYears(timestamp);
71+
case MONTHS:
72+
return DateTimeUtil.microsToMonths(timestamp);
73+
case DAYS:
74+
return DateTimeUtil.microsToDays(timestamp);
75+
case HOURS:
76+
return DateTimeUtil.microsToHours(timestamp);
77+
default:
78+
throw new UnsupportedOperationException("Unsupported time unit: " + granularity);
79+
}
80+
case NANOS:
81+
switch (granularity) {
82+
case YEARS:
83+
return DateTimeUtil.nanosToYears(timestamp);
84+
case MONTHS:
85+
return DateTimeUtil.nanosToMonths(timestamp);
86+
case DAYS:
87+
return DateTimeUtil.nanosToDays(timestamp);
88+
case HOURS:
89+
return DateTimeUtil.nanosToHours(timestamp);
90+
default:
91+
throw new UnsupportedOperationException("Unsupported time unit: " + granularity);
92+
}
93+
default:
94+
throw new UnsupportedOperationException("Unsupported time unit: " + timestampUnit);
95+
}
96+
}
97+
}
4498

4599
private final ChronoUnit granularity;
46100
private final String name;
47-
private final SerializableFunction<Long, Integer> apply;
101+
private final Apply apply;
48102

49-
Timestamps(ChronoUnit granularity, String name, SerializableFunction<Long, Integer> apply) {
103+
Timestamps(ChronoUnit granularity, String name, TimestampUnit timestampUnit) {
50104
this.name = name;
51105
this.granularity = granularity;
52-
this.apply = apply;
106+
this.apply = new Apply(granularity, timestampUnit);
53107
}
54108

55109
/**
@@ -185,116 +239,4 @@ public String toString() {
185239
public String dedupName() {
186240
return "time";
187241
}
188-
189-
@Immutable
190-
static class MicrosToYears implements SerializableFunction<Long, Integer> {
191-
static final MicrosToYears INSTANCE = new MicrosToYears();
192-
193-
@Override
194-
public Integer apply(Long micros) {
195-
if (micros == null) {
196-
return null;
197-
}
198-
199-
return DateTimeUtil.microsToYears(micros);
200-
}
201-
}
202-
203-
@Immutable
204-
static class MicrosToMonths implements SerializableFunction<Long, Integer> {
205-
static final MicrosToMonths INSTANCE = new MicrosToMonths();
206-
207-
@Override
208-
public Integer apply(Long micros) {
209-
if (micros == null) {
210-
return null;
211-
}
212-
213-
return DateTimeUtil.microsToMonths(micros);
214-
}
215-
}
216-
217-
@Immutable
218-
static class MicrosToDays implements SerializableFunction<Long, Integer> {
219-
static final MicrosToDays INSTANCE = new MicrosToDays();
220-
221-
@Override
222-
public Integer apply(Long micros) {
223-
if (micros == null) {
224-
return null;
225-
}
226-
227-
return DateTimeUtil.microsToDays(micros);
228-
}
229-
}
230-
231-
@Immutable
232-
static class MicrosToHours implements SerializableFunction<Long, Integer> {
233-
static final MicrosToHours INSTANCE = new MicrosToHours();
234-
235-
@Override
236-
public Integer apply(Long micros) {
237-
if (micros == null) {
238-
return null;
239-
}
240-
241-
return DateTimeUtil.microsToHours(micros);
242-
}
243-
}
244-
245-
@Immutable
246-
static class NanosToYears implements SerializableFunction<Long, Integer> {
247-
static final NanosToYears INSTANCE = new NanosToYears();
248-
249-
@Override
250-
public Integer apply(Long nanos) {
251-
if (nanos == null) {
252-
return null;
253-
}
254-
255-
return DateTimeUtil.nanosToYears(nanos);
256-
}
257-
}
258-
259-
@Immutable
260-
static class NanosToMonths implements SerializableFunction<Long, Integer> {
261-
static final NanosToMonths INSTANCE = new NanosToMonths();
262-
263-
@Override
264-
public Integer apply(Long nanos) {
265-
if (nanos == null) {
266-
return null;
267-
}
268-
269-
return DateTimeUtil.nanosToMonths(nanos);
270-
}
271-
}
272-
273-
@Immutable
274-
static class NanosToDays implements SerializableFunction<Long, Integer> {
275-
static final NanosToDays INSTANCE = new NanosToDays();
276-
277-
@Override
278-
public Integer apply(Long nanos) {
279-
if (nanos == null) {
280-
return null;
281-
}
282-
283-
return DateTimeUtil.nanosToDays(nanos);
284-
}
285-
}
286-
287-
@Immutable
288-
static class NanosToHours implements SerializableFunction<Long, Integer> {
289-
static final NanosToHours INSTANCE = new NanosToHours();
290-
291-
@Override
292-
public Integer apply(Long nanos) {
293-
if (nanos == null) {
294-
return null;
295-
}
296-
297-
return DateTimeUtil.nanosToHours(nanos);
298-
}
299-
}
300242
}

azure/src/main/java/org/apache/iceberg/azure/adlsv2/ADLSFileIO.java

Lines changed: 1 addition & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -133,7 +133,7 @@ public void initialize(Map<String, String> props) {
133133
initMetrics(properties);
134134
this.azureProperties
135135
.vendedAdlsCredentialProvider()
136-
.ifPresent((provider -> this.vendedAdlsCredentialProvider = provider));
136+
.ifPresent(provider -> this.vendedAdlsCredentialProvider = provider);
137137
}
138138

139139
@SuppressWarnings("CatchBlockLogException")

bigquery/src/main/java/org/apache/iceberg/gcp/bigquery/BigQueryMetastoreClientImpl.java

Lines changed: 25 additions & 14 deletions
Original file line numberDiff line numberDiff line change
@@ -50,6 +50,7 @@
5050
import com.google.cloud.bigquery.BigQueryRetryHelper;
5151
import java.io.IOException;
5252
import java.security.GeneralSecurityException;
53+
import java.util.Collections;
5354
import java.util.List;
5455
import java.util.Locale;
5556
import java.util.Map;
@@ -73,6 +74,8 @@
7374
import org.apache.iceberg.relocated.com.google.common.base.Preconditions;
7475
import org.apache.iceberg.relocated.com.google.common.collect.Lists;
7576
import org.apache.iceberg.relocated.com.google.common.collect.Maps;
77+
import org.apache.iceberg.util.Tasks;
78+
import org.apache.iceberg.util.ThreadPools;
7679

7780
/** A client of Google Bigquery Metastore functions over the BigQuery service. */
7881
public final class BigQueryMetastoreClientImpl implements BigQueryMetastoreClient {
@@ -485,22 +488,30 @@ public List<Tables> list(DatasetReference datasetReference, boolean listAllTable
485488
// distinguish Iceberg
486489
// tables for us to filter out those results since invoking `getTable` on them would
487490
// correctly raise a `NoSuchIcebergTableException` for being inoperable by this plugin.
491+
492+
List<Tables> allTables = tablesStream.collect(Collectors.toList());
493+
488494
if (!listAllTables) {
489-
tablesStream =
490-
tablesStream
491-
.parallel()
492-
.filter(
493-
table -> {
494-
try {
495-
load(table.getTableReference());
496-
} catch (NoSuchTableException e) {
497-
return false;
498-
}
499-
return true;
500-
});
495+
List<Tables> validTables = Collections.synchronizedList(Lists.newArrayList());
496+
Tasks.foreach(allTables)
497+
.executeWith(ThreadPools.getWorkerPool())
498+
.noRetry()
499+
.suppressFailureWhenFinished()
500+
.run(
501+
table -> {
502+
try {
503+
load(table.getTableReference());
504+
validTables.add(table);
505+
} catch (NoSuchTableException e) {
506+
// Silently ignore tables that are not valid Iceberg tables
507+
// This is expected behavior as we're filtering out non-Iceberg tables
508+
}
509+
});
510+
511+
return validTables;
512+
} else {
513+
return allTables;
501514
}
502-
503-
return tablesStream.collect(Collectors.toList());
504515
} catch (IOException e) {
505516
throw new RuntimeIOException("%s", e);
506517
}

flink/v1.19/flink/src/main/java/org/apache/iceberg/flink/sink/dynamic/DynamicRecordInternalSerializer.java

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,6 @@
2020

2121
import java.io.IOException;
2222
import java.util.Collections;
23-
import java.util.Objects;
2423
import java.util.Set;
2524
import org.apache.flink.annotation.Internal;
2625
import org.apache.flink.api.common.typeutils.TypeSerializer;
@@ -231,7 +230,7 @@ public boolean equals(Object obj) {
231230

232231
@Override
233232
public int hashCode() {
234-
return Objects.hashCode(writeSchemaAndSpec);
233+
return Boolean.hashCode(writeSchemaAndSpec);
235234
}
236235

237236
@Override

flink/v1.20/flink/src/main/java/org/apache/iceberg/flink/sink/dynamic/DynamicRecordInternalSerializer.java

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,6 @@
2020

2121
import java.io.IOException;
2222
import java.util.Collections;
23-
import java.util.Objects;
2423
import java.util.Set;
2524
import org.apache.flink.annotation.Internal;
2625
import org.apache.flink.api.common.typeutils.TypeSerializer;
@@ -231,7 +230,7 @@ public boolean equals(Object obj) {
231230

232231
@Override
233232
public int hashCode() {
234-
return Objects.hashCode(writeSchemaAndSpec);
233+
return Boolean.hashCode(writeSchemaAndSpec);
235234
}
236235

237236
@Override

flink/v2.0/flink/src/main/java/org/apache/iceberg/flink/sink/dynamic/DynamicRecordInternalSerializer.java

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,6 @@
2020

2121
import java.io.IOException;
2222
import java.util.Collections;
23-
import java.util.Objects;
2423
import java.util.Set;
2524
import org.apache.flink.annotation.Internal;
2625
import org.apache.flink.api.common.typeutils.TypeSerializer;
@@ -231,7 +230,7 @@ public boolean equals(Object obj) {
231230

232231
@Override
233232
public int hashCode() {
234-
return Objects.hashCode(writeSchemaAndSpec);
233+
return Boolean.hashCode(writeSchemaAndSpec);
235234
}
236235

237236
@Override

0 commit comments

Comments
 (0)