Fix sonar warnings (#181)

pull/191/head
Michael Barry 2022-04-23 05:58:49 -04:00 zatwierdzone przez GitHub
rodzic ae92610f24
commit 5341d4d712
Nie znaleziono w bazie danych klucza dla tego podpisu
ID klucza GPG: 4AEE18F83AFDEB23
41 zmienionych plików z 359 dodań i 220 usunięć

Wyświetl plik

@ -105,7 +105,7 @@ public class Generate {
}
private static <T> T loadAndParseYaml(String url, PlanetilerConfig config, Class<T> clazz) throws IOException {
LOGGER.info("reading " + url);
LOGGER.info("reading {}", url);
try (var stream = Downloader.openStream(url, config)) {
// Jackson yaml parsing does not handle anchors and references, so first parse the input
// using SnakeYAML, then parse SnakeYAML's output using Jackson to get it into our records.
@ -150,7 +150,7 @@ public class Generate {
String mappingPath = Path.of(layerFile).resolveSibling(datasource.mapping_file).normalize().toString();
imposm3MappingFiles.add(base + mappingPath);
} else {
LOGGER.warn("Unknown datasource type: " + datasource.type);
LOGGER.warn("Unknown datasource type: {}", datasource.type);
}
}
}
@ -675,7 +675,7 @@ public class Generate {
return Stream.of(markdown.strip().split("[\r\n][\r\n]+"))
.map(p -> parser.parse(p.strip()))
.map(node -> escapeJavadoc(renderer.render(node)))
.map(p -> p.replaceAll("(^<p>|</p>$)", "").strip())
.map(p -> p.replaceAll("((^<p>)|(</p>$))", "").strip())
.collect(joining(LINE_SEPARATOR + "<p>" + LINE_SEPARATOR));
}

Wyświetl plik

@ -82,7 +82,7 @@ public class MountainPeak implements
* label density by only taking the top 5 most important mountain peaks within each 100x100px
* square.
*/
private static final Logger LOGGER = LoggerFactory.getLogger(TransportationName.class);
private static final Logger LOGGER = LoggerFactory.getLogger(MountainPeak.class);
private final Translations translations;
private final Stats stats;

Wyświetl plik

@ -235,7 +235,7 @@ public class Place implements
rank = country.rank;
}
rank = Math.min(6, Math.max(1, rank));
rank = Math.max(1, Math.min(6, rank));
features.point(LAYER_NAME).setBufferPixels(BUFFER_SIZE)
.putAttrs(names)

Wyświetl plik

@ -106,7 +106,7 @@ public class WaterName implements
// TODO pull lake centerline computation into planetiler?
long osmId = Math.abs(feature.getLong("OSM_ID"));
if (osmId == 0L) {
LOGGER.warn("Bad lake centerline. Tags: " + feature.tags());
LOGGER.warn("Bad lake centerline. Tags: {}", feature.tags());
} else {
try {
// multiple threads call this concurrently
@ -179,7 +179,7 @@ public class WaterName implements
if (centerlineGeometry != null) {
// prefer lake centerline if it exists
feature = features.geometry(LAYER_NAME, centerlineGeometry)
.setMinPixelSizeBelowZoom(13, 6 * element.name().length());
.setMinPixelSizeBelowZoom(13, 6d * element.name().length());
} else {
// otherwise just use a label point inside the lake
feature = features.pointOnSurface(LAYER_NAME);

Wyświetl plik

@ -54,28 +54,26 @@ import java.util.stream.Stream;
* <a href="https://github.com/openmaptiles/openmaptiles-tools/blob/master/sql/zzz_language.sql">openmaptiles-tools</a>.
*/
public class LanguageUtils {
// See https://github.com/onthegomap/planetiler/issues/86
// Name tags that should be eligible for finding a latin name.
// See https://wiki.openstreetmap.org/wiki/Multilingual_names
private static final Predicate<String> VALID_NAME_TAGS =
Pattern
.compile("^name:[a-z]{2,3}(-[a-z]{4})?([-_](x-)?[a-z]{2,})?(-([a-z]{2}|[0-9]{3}))?$", Pattern.CASE_INSENSITIVE)
.asMatchPredicate();
// See https://github.com/onthegomap/planetiler/issues/86
// Match strings that only contain latin characters.
private static final Predicate<String> ONLY_LATIN = Pattern
.compile("^[\\P{IsLetter}[\\p{IsLetter}&&\\p{IsLatin}]]+$")
.asMatchPredicate();
// Match only latin letters
private static final Pattern LATIN_LETTER = Pattern.compile("[\\p{IsLetter}&&\\p{IsLatin}]+");
private static final Pattern EMPTY_PARENS = Pattern.compile("(\\([ -.]*\\)|\\[[ -.]*])");
private static final Pattern LEADING_TRAILING_JUNK = Pattern.compile("(^\\s*([./-]\\s*)*|(\\s+[./-])*\\s*$)");
private static final Pattern LEADING_TRAILING_JUNK = Pattern.compile("((^[\\s./-]*)|([\\s./-]*$))");
private static final Pattern WHITESPACE = Pattern.compile("\\s+");
private static final Set<String> EN_DE_NAME_KEYS = Set.of("name:en", "name:de");
private LanguageUtils() {}
private static void putIfNotEmpty(Map<String, Object> dest, String key, Object value) {
if (value != null && !value.equals("")) {
dest.put(key, value);

Wyświetl plik

@ -39,8 +39,8 @@ interface AppendStore extends Closeable, MemoryEstimator.HasEstimate, DiskBacked
static Ints create(Storage storage, Storage.Params params) {
return switch (storage) {
case DIRECT -> new AppendStoreRam.Ints(true, params);
case RAM -> new AppendStoreRam.Ints(false, params);
case DIRECT -> new AppendStoreRam.Ints(true);
case RAM -> new AppendStoreRam.Ints(false);
case MMAP -> new AppendStoreMmap.Ints(params);
};
}

Wyświetl plik

@ -30,8 +30,8 @@ abstract class AppendStoreMmap implements AppendStore {
private final Path path;
private final boolean madvise;
long outIdx = 0;
private volatile MappedByteBuffer[] segments;
private volatile FileChannel channel;
private volatile MappedByteBuffer[] segments; // NOSONAR - array is not thread-safe, but it's immutable after creation
private volatile FileChannel channel; // NOSONAR - channel is not thread-safe, but we only map over it once then close
AppendStoreMmap(Path path, boolean madvise) {
this(path, 1 << 30, madvise); // 1GB
@ -83,7 +83,7 @@ abstract class AppendStoreMmap implements AppendStore {
try {
ByteBufferUtil.free(segments);
} catch (IOException e) {
LOGGER.info("Unable to unmap " + path + " " + e);
LOGGER.info("Unable to unmap {} {}", path, e);
}
Arrays.fill(segments, null);
}

Wyświetl plik

@ -33,7 +33,7 @@ abstract class AppendStoreRam implements AppendStore {
throw new IllegalStateException("Segment size must be a multiple of 8: " + segmentSizeBytes);
}
this.slabSize = (1 << slabBits);
this.slabMask = slabSize - 1;
this.slabMask = slabSize - 1L;
this.arrays = new ArrayList<>();
}
@ -57,10 +57,6 @@ abstract class AppendStoreRam implements AppendStore {
static class Ints extends AppendStoreRam implements AppendStore.Ints {
Ints(boolean direct, Storage.Params params) {
this(direct);
}
Ints(boolean direct) {
this(direct, 1 << 20); // 1MB
}

Wyświetl plik

@ -1,5 +1,7 @@
package com.onthegomap.planetiler.collection;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import com.onthegomap.planetiler.config.PlanetilerConfig;
import com.onthegomap.planetiler.stats.ProcessInfo;
import com.onthegomap.planetiler.stats.ProgressLoggers;
@ -21,6 +23,7 @@ import java.util.ArrayList;
import java.util.Arrays;
import java.util.Iterator;
import java.util.List;
import java.util.NoSuchElementException;
import java.util.PriorityQueue;
import java.util.concurrent.Semaphore;
import java.util.concurrent.atomic.AtomicLong;
@ -45,7 +48,7 @@ import org.slf4j.LoggerFactory;
@NotThreadSafe
class ExternalMergeSort implements FeatureSort {
private static final Logger LOGGER = LoggerFactory.getLogger(FeatureSort.class);
private static final Logger LOGGER = LoggerFactory.getLogger(ExternalMergeSort.class);
private static final long MAX_CHUNK_SIZE = 1_000_000_000; // 1GB
private final Path dir;
private final Stats stats;
@ -88,7 +91,7 @@ class ExternalMergeSort implements FeatureSort {
this.workers = workers;
this.readerLimit = Math.max(1, config.sortMaxReaders());
this.writerLimit = Math.max(1, config.sortMaxWriters());
LOGGER.info("Using merge sort feature map, chunk size=" + (chunkSizeLimit / 1_000_000) + "mb workers=" + workers);
LOGGER.info("Using merge sort feature map, chunk size={}mb workers={}", chunkSizeLimit / 1_000_000, workers);
try {
FileUtils.deleteDirectory(dir);
Files.createDirectories(dir);
@ -181,7 +184,8 @@ class ExternalMergeSort implements FeatureSort {
doneCounter.incrementAndGet();
} catch (InterruptedException e) {
throw new RuntimeException(e);
Thread.currentThread().interrupt();
throwFatalException(e);
}
});
@ -197,9 +201,10 @@ class ExternalMergeSort implements FeatureSort {
sorted = true;
timer.stop();
LOGGER.info("read:" + Duration.ofNanos(reading.get()).toSeconds() +
"s write:" + Duration.ofNanos(writing.get()).toSeconds() +
"s sort:" + Duration.ofNanos(sorting.get()).toSeconds() + "s");
LOGGER.info("read:{}s write:{}s sort:{}s",
Duration.ofNanos(reading.get()).toSeconds(),
Duration.ofNanos(writing.get()).toSeconds(),
Duration.ofNanos(sorting.get()).toSeconds());
}
@Override
@ -379,6 +384,9 @@ class ExternalMergeSort implements FeatureSort {
@Override
public SortableFeature next() {
SortableFeature current = next;
if (current == null) {
throw new NoSuchElementException();
}
if ((next = readNextFeature()) == null) {
close();
}

Wyświetl plik

@ -225,52 +225,6 @@ public final class FeatureGroup implements Consumer<SortableFeature>, Iterable<F
return packer.toByteArray();
}
private VectorTile.Feature decodeVectorTileFeature(SortableFeature entry) {
try (MessageUnpacker unpacker = MessagePack.newDefaultUnpacker(entry.value())) {
long group;
if (extractHasGroupFromKey(entry.key())) {
group = unpacker.unpackLong();
unpacker.unpackInt(); // groupLimit - features over the limit were already discarded
} else {
group = VectorTile.Feature.NO_GROUP;
}
long id = unpacker.unpackLong();
byte geomTypeAndScale = unpacker.unpackByte();
GeometryType geomType = decodeGeomType(geomTypeAndScale);
int scale = decodeScale(geomTypeAndScale);
int mapSize = unpacker.unpackMapHeader();
Map<String, Object> attrs = new HashMap<>(mapSize);
for (int i = 0; i < mapSize; i++) {
String key = commonStrings.decode(unpacker.unpackByte());
Value v = unpacker.unpackValue();
if (v.isStringValue()) {
attrs.put(key, v.asStringValue().asString());
} else if (v.isIntegerValue()) {
attrs.put(key, v.asIntegerValue().toLong());
} else if (v.isFloatValue()) {
attrs.put(key, v.asFloatValue().toDouble());
} else if (v.isBooleanValue()) {
attrs.put(key, v.asBooleanValue().getBoolean());
}
}
int commandSize = unpacker.unpackArrayHeader();
int[] commands = new int[commandSize];
for (int i = 0; i < commandSize; i++) {
commands[i] = unpacker.unpackInt();
}
String layer = commonStrings.decode(extractLayerIdFromKey(entry.key()));
return new VectorTile.Feature(
layer,
id,
new VectorTile.VectorGeometry(commands, geomType, scale),
attrs,
group
);
} catch (IOException e) {
throw new IllegalStateException(e);
}
}
static GeometryType decodeGeomType(byte geomTypeAndScale) {
return GeometryType.valueOf((byte) (geomTypeAndScale & 0b111));
}
@ -282,7 +236,7 @@ public final class FeatureGroup implements Consumer<SortableFeature>, Iterable<F
static byte encodeGeomTypeAndScale(VectorTile.VectorGeometry geometry) {
assert geometry.geomType().asByte() >= 0 && geometry.geomType().asByte() <= 8;
assert geometry.scale() >= 0 && geometry.scale() < (1 << 5);
return (byte) (geometry.geomType().asByte() | (geometry.scale() << 3));
return (byte) ((geometry.geomType().asByte() & 0xff) | (geometry.scale() << 3));
}
/** Writes a serialized binary feature to intermediate storage. */
@ -361,7 +315,7 @@ public final class FeatureGroup implements Consumer<SortableFeature>, Iterable<F
private final List<SortableFeature> entries = new ArrayList<>();
private final AtomicLong numFeaturesProcessed = new AtomicLong(0);
private LongLongHashMap counts = null;
private byte layer = Byte.MAX_VALUE;
private byte lastLayer = Byte.MAX_VALUE;
private TileFeatures(int tileCoord) {
this.tileCoord = TileCoord.decode(tileCoord);
@ -403,6 +357,52 @@ public final class FeatureGroup implements Consumer<SortableFeature>, Iterable<F
return true;
}
private VectorTile.Feature decodeVectorTileFeature(SortableFeature entry) {
try (MessageUnpacker unpacker = MessagePack.newDefaultUnpacker(entry.value())) {
long group;
if (extractHasGroupFromKey(entry.key())) {
group = unpacker.unpackLong();
unpacker.unpackInt(); // groupLimit - features over the limit were already discarded
} else {
group = VectorTile.Feature.NO_GROUP;
}
long id = unpacker.unpackLong();
byte geomTypeAndScale = unpacker.unpackByte();
GeometryType geomType = decodeGeomType(geomTypeAndScale);
int scale = decodeScale(geomTypeAndScale);
int mapSize = unpacker.unpackMapHeader();
Map<String, Object> attrs = new HashMap<>(mapSize);
for (int i = 0; i < mapSize; i++) {
String key = commonStrings.decode(unpacker.unpackByte());
Value v = unpacker.unpackValue();
if (v.isStringValue()) {
attrs.put(key, v.asStringValue().asString());
} else if (v.isIntegerValue()) {
attrs.put(key, v.asIntegerValue().toLong());
} else if (v.isFloatValue()) {
attrs.put(key, v.asFloatValue().toDouble());
} else if (v.isBooleanValue()) {
attrs.put(key, v.asBooleanValue().getBoolean());
}
}
int commandSize = unpacker.unpackArrayHeader();
int[] commands = new int[commandSize];
for (int i = 0; i < commandSize; i++) {
commands[i] = unpacker.unpackInt();
}
String layer = commonStrings.decode(extractLayerIdFromKey(entry.key()));
return new VectorTile.Feature(
layer,
id,
new VectorTile.VectorGeometry(commands, geomType, scale),
attrs,
group
);
} catch (IOException e) {
throw new IllegalStateException(e);
}
}
public VectorTile getVectorTileEncoder() {
VectorTile encoder = new VectorTile();
List<VectorTile.Feature> items = new ArrayList<>(entries.size());
@ -447,19 +447,17 @@ public final class FeatureGroup implements Consumer<SortableFeature>, Iterable<F
// introduce artificial intersections between endpoints to confuse line merging,
// so we have to reduce the precision here, now that line merging is done.
unscale(features);
} catch (Throwable e) {
} catch (Throwable e) { // NOSONAR - OK to catch Throwable since we re-throw Errors
// failures in tile post-processing happen very late so err on the side of caution and
// log failures, only throwing when it's a fatal error
if (e instanceof GeometryException geoe) {
geoe.log(stats, "postprocess_layer",
"Caught error postprocessing features for " + layer + " layer on " + tileCoord);
} else if (e instanceof Exception) {
LOGGER.error("Caught error postprocessing features " + layer + " " + tileCoord, e);
} else {
LOGGER.error("Fatal error postprocessing features " + layer + " " + tileCoord, e);
}
if (e instanceof Error err) {
} else if (e instanceof Error err) {
LOGGER.error("Caught fatal error postprocessing features {} {}", layer, tileCoord, e);
throw err;
} else {
LOGGER.error("Caught error postprocessing features {} {}", layer, tileCoord, e);
}
}
encoder.addLayerFeatures(layer, features);
@ -472,9 +470,9 @@ public final class FeatureGroup implements Consumer<SortableFeature>, Iterable<F
byte thisLayer = extractLayerIdFromKey(key);
if (counts == null) {
counts = Hppc.newLongLongHashMap();
layer = thisLayer;
} else if (thisLayer != layer) {
layer = thisLayer;
lastLayer = thisLayer;
} else if (thisLayer != lastLayer) {
lastLayer = thisLayer;
counts.clear();
}
var groupInfo = peekAtGroupInfo(entry.value());

Wyświetl plik

@ -1,6 +1,7 @@
package com.onthegomap.planetiler.collection;
import java.util.Iterator;
import java.util.NoSuchElementException;
import java.util.function.Supplier;
/**
@ -32,7 +33,9 @@ public interface IterableOnce<T> extends Iterable<T>, Supplier<T> {
@Override
public T next() {
advance();
if (!hasNext()) {
throw new NoSuchElementException();
}
stale = true;
return next;
}

Wyświetl plik

@ -101,7 +101,9 @@ public interface LongLongMultimap extends MemoryEstimator.HasEstimate, DiskBacke
class Noop implements Replaceable, Appendable {
@Override
public void put(long key, long value) {}
public void put(long key, long value) {
// do nothing on update
}
@Override
public LongArrayList get(long key) {
@ -114,10 +116,14 @@ public interface LongLongMultimap extends MemoryEstimator.HasEstimate, DiskBacke
}
@Override
public void close() {}
public void close() {
// nothing to close
}
@Override
public void replaceValues(long key, LongArrayList values) {}
public void replaceValues(long key, LongArrayList values) {
// do nothing on update
}
}
/**
@ -191,7 +197,7 @@ public interface LongLongMultimap extends MemoryEstimator.HasEstimate, DiskBacke
}
keys.buffer = sortedKeys;
values.buffer = sortedValues;
LOGGER.debug("Sorted long long multimap " + timer.stop());
LOGGER.debug("Sorted long long multimap {}", timer.stop());
}
@Override
@ -266,7 +272,7 @@ public interface LongLongMultimap extends MemoryEstimator.HasEstimate, DiskBacke
LongArrayList result = new LongArrayList();
int num = (int) values.getLong(index);
for (int i = 0; i < num; i++) {
result.add(values.getLong(i + index + 1));
result.add(values.getLong(i + index + 1L));
}
return result;
} else {

Wyświetl plik

@ -62,7 +62,7 @@ public class SparseArrayLongLongMap implements LongLongMap, LongLongMap.Sequenti
}
long lo = offsets.getLong(chunk);
long hi = Math.min(values.size(), chunk >= offsets.size() - 1 ? values.size() : offsets.getLong(chunk + 1)) - 1;
long hi = Math.min(values.size(), chunk >= offsets.size() - 1 ? values.size() : offsets.getLong(chunk + 1L)) - 1;
int startPad = offsetStartPad.get(chunk) & 255;
long index = lo + offset - startPad;

Wyświetl plik

@ -231,7 +231,8 @@ public class Arguments {
/** Returns a {@link List} parsed from {@code key} argument where values are separated by commas. */
public List<String> getList(String key, String description, List<String> defaultValue) {
String value = getArg(key, String.join(",", defaultValue));
List<String> results = Stream.of(value.split("\\s*,[,\\s]*"))
List<String> results = Stream.of(value.split(","))
.map(String::trim)
.filter(c -> !c.isBlank()).toList();
logArgValue(key, description, value);
return results;

Wyświetl plik

@ -196,7 +196,7 @@ public class GeoUtils {
* {@link #encodeFlatLocation(double, double)}.
*/
public static double decodeWorldY(long encoded) {
return (((double) (encoded & LOWER_32_BIT_MASK)) / HALF_QUANTIZED_WORLD_SIZE) - 1;
return ((encoded & LOWER_32_BIT_MASK) / HALF_QUANTIZED_WORLD_SIZE) - 1;
}
/**
@ -204,7 +204,7 @@ public class GeoUtils {
* {@link #encodeFlatLocation(double, double)}.
*/
public static double decodeWorldX(long encoded) {
return (((double) (encoded >>> 32)) / HALF_QUANTIZED_WORLD_SIZE) - 1;
return ((encoded >>> 32) / HALF_QUANTIZED_WORLD_SIZE) - 1;
}
/**
@ -227,7 +227,7 @@ public class GeoUtils {
/** Returns the width in meters of a single pixel of a 256x256 px tile at the given {@code zoom} level. */
public static double metersPerPixelAtEquator(int zoom) {
return WORLD_CIRCUMFERENCE_METERS / Math.pow(2, zoom + 8);
return WORLD_CIRCUMFERENCE_METERS / Math.pow(2d, zoom + 8d);
}
/** Returns the length in pixels for a given number of meters on a 256x256 px tile at the given {@code zoom} level. */
@ -321,7 +321,7 @@ public class GeoUtils {
private static long longPair(int a, int b) {
return (((long) a) << 32L) | (((long) b) & LOWER_32_BIT_MASK);
return (((long) a) << 32L) | (b & LOWER_32_BIT_MASK);
}
/**
@ -356,7 +356,7 @@ public class GeoUtils {
public static Geometry polygonToLineString(Geometry geom) throws GeometryException {
List<LineString> lineStrings = new ArrayList<>();
getLineStrings(geom, lineStrings);
if (lineStrings.size() == 0) {
if (lineStrings.isEmpty()) {
throw new GeometryException("polygon_to_linestring_empty", "No line strings");
} else if (lineStrings.size() == 1) {
return lineStrings.get(0);

Wyświetl plik

@ -370,15 +370,21 @@ public final class Mbtiles implements Closeable {
batchStatement = createBatchStatement(batchLimit);
}
@SuppressWarnings("java:S2077")
private PreparedStatement createBatchStatement(int size) {
List<String> groups = new ArrayList<>();
for (int i = 0; i < size; i++) {
groups.add("(?,?,?,?)");
}
try {
return connection.prepareStatement(
"INSERT INTO " + TILES_TABLE + " (" + TILES_COL_Z + "," + TILES_COL_X + "," + TILES_COL_Y + "," +
TILES_COL_DATA + ") VALUES " + String.join(", ", groups) + ";");
return connection.prepareStatement("""
INSERT INTO %s (%s, %s, %s, %s) VALUES %s;
""".formatted(
TILES_TABLE,
TILES_COL_Z, TILES_COL_X, TILES_COL_Y,
TILES_COL_DATA,
String.join(", ", groups)
));
} catch (SQLException throwables) {
throw new IllegalStateException("Could not create prepared statement", throwables);
}

Wyświetl plik

@ -197,7 +197,7 @@ public class MbtilesWriter {
for (var feature : features) {
int z = feature.tileCoord().z();
if (z != currentZoom) {
LOGGER.trace("Starting z" + z);
LOGGER.trace("Starting z{}", z);
currentZoom = z;
}
long thisTileFeatures = feature.getNumFeaturesToEmit();
@ -251,11 +251,13 @@ public class MbtilesWriter {
lastEncoded = encoded;
lastBytes = bytes;
if (encoded.length > 1_000_000) {
LOGGER.warn(tileFeatures.tileCoord() + " " + (encoded.length / 1024) + "kb uncompressed");
LOGGER.warn("{} {}kb uncompressed",
tileFeatures.tileCoord(),
encoded.length / 1024);
}
}
int zoom = tileFeatures.tileCoord().z();
int encodedLength = encoded.length;
int encodedLength = encoded == null ? 0 : encoded.length;
totalTileSizesByZoom[zoom].incBy(encodedLength);
maxTileSizesByZoom[zoom].accumulate(encodedLength);
result.add(new Mbtiles.TileEntry(tileFeatures.tileCoord(), bytes));
@ -271,7 +273,7 @@ public class MbtilesWriter {
if (!config.deferIndexCreation()) {
db.addTileIndex();
} else {
LOGGER.info("Deferring index creation. Add later by executing: " + Mbtiles.ADD_TILE_INDEX_SQL);
LOGGER.info("Deferring index creation. Add later by executing: {}", Mbtiles.ADD_TILE_INDEX_SQL);
}
db.metadata()
@ -301,9 +303,9 @@ public class MbtilesWriter {
int z = tileCoord.z();
if (z != currentZ) {
if (time == null) {
LOGGER.info("Starting z" + z);
LOGGER.info("Starting z{}", z);
} else {
LOGGER.info("Finished z" + currentZ + " in " + time.stop() + ", now starting z" + z);
LOGGER.info("Finished z{} in {}, now starting z{}", currentZ, time.stop(), z);
}
time = Timer.start();
currentZ = z;
@ -317,7 +319,7 @@ public class MbtilesWriter {
}
if (time != null) {
LOGGER.info("Finished z" + currentZ + " in " + time.stop());
LOGGER.info("Finished z{} in {}", currentZ, time.stop());
}
if (config.optimizeDb()) {
@ -337,15 +339,16 @@ public class MbtilesWriter {
sumSize += totalSize;
sumCount += totalCount;
long maxSize = maxTileSizesByZoom[z].get();
LOGGER.debug("z" + z +
" avg:" + format.storage(totalSize / Math.max(totalCount, 1), false) +
" max:" + format.storage(maxSize, false));
LOGGER.debug("z{} avg:{} max:{}",
z,
format.storage(totalCount == 0 ? 0 : (totalSize / totalCount), false),
format.storage(maxSize, false));
}
LOGGER.debug("all" +
" avg:" + format.storage(sumSize / Math.max(sumCount, 1), false) +
" max:" + format.storage(maxMax, false));
LOGGER.debug(" # features: " + format.integer(featuresProcessed.get()));
LOGGER.debug(" # tiles: " + format.integer(this.tilesEmitted()));
LOGGER.debug("all avg:{} max:{}",
format.storage(sumCount == 0 ? 0 : (sumSize / sumCount), false),
format.storage(maxMax, false));
LOGGER.debug(" # features: {}", format.integer(featuresProcessed.get()));
LOGGER.debug(" # tiles: {}", format.integer(this.tilesEmitted()));
}
private long tilesEmitted() {

Wyświetl plik

@ -0,0 +1,14 @@
package com.onthegomap.planetiler.reader;
/**
* Error encountered while parsing an input file.
*/
public class FileFormatException extends RuntimeException {
public FileFormatException(String message) {
super(message);
}
public FileFormatException(String message, Throwable throwable) {
super(message, throwable);
}
}

Wyświetl plik

@ -1,5 +1,7 @@
package com.onthegomap.planetiler.reader;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import com.onthegomap.planetiler.Profile;
import com.onthegomap.planetiler.collection.FeatureGroup;
import com.onthegomap.planetiler.config.PlanetilerConfig;
@ -21,6 +23,7 @@ import java.sql.Statement;
import java.util.ArrayList;
import java.util.HashMap;
import java.util.List;
import java.util.regex.Pattern;
import org.locationtech.jts.geom.Geometry;
import org.slf4j.Logger;
import org.slf4j.LoggerFactory;
@ -33,6 +36,7 @@ import org.slf4j.LoggerFactory;
*/
public class NaturalEarthReader extends SimpleReader {
private static final Pattern VALID_TABLE_NAME = Pattern.compile("ne_[a-z0-9_]+", Pattern.CASE_INSENSITIVE);
private static final Logger LOGGER = LoggerFactory.getLogger(NaturalEarthReader.class);
private final Connection conn;
private Path extracted;
@ -80,19 +84,18 @@ public class NaturalEarthReader extends SimpleReader {
private Connection open(Path path, Path tmpLocation) throws IOException, SQLException {
String uri = "jdbc:sqlite:" + path.toAbsolutePath();
if (FileUtils.hasExtension(path, "zip")) {
Path toOpen = tmpLocation == null ? Files.createTempFile("sqlite", "natearth") : tmpLocation;
extracted = toOpen;
extracted = tmpLocation;
try (var zipFs = FileSystems.newFileSystem(path)) {
var zipEntry = FileUtils.walkFileSystem(zipFs)
.filter(Files::isRegularFile)
.filter(entry -> FileUtils.hasExtension(entry, "sqlite"))
.findFirst()
.orElseThrow(() -> new IllegalArgumentException("No .sqlite file found inside " + path));
LOGGER.info("unzipping " + path.toAbsolutePath() + " to " + extracted);
LOGGER.info("unzipping {} to {}", path.toAbsolutePath(), extracted);
Files.copy(Files.newInputStream(zipEntry), extracted, StandardCopyOption.REPLACE_EXISTING);
extracted.toFile().deleteOnExit();
}
uri = "jdbc:sqlite:" + toOpen.toAbsolutePath();
uri = "jdbc:sqlite:" + tmpLocation.toAbsolutePath();
}
return DriverManager.getConnection(uri);
}
@ -102,10 +105,12 @@ public class NaturalEarthReader extends SimpleReader {
try (ResultSet rs = conn.getMetaData().getTables(null, null, null, null)) {
while (rs.next()) {
String table = rs.getString("TABLE_NAME");
result.add(table);
if (VALID_TABLE_NAME.matcher(table).matches()) {
result.add(table);
}
}
} catch (SQLException e) {
throw new RuntimeException(e);
throwFatalException(e);
}
return result;
}
@ -116,7 +121,8 @@ public class NaturalEarthReader extends SimpleReader {
for (String table : tableNames()) {
try (
var stmt = conn.createStatement();
var result = stmt.executeQuery("select count(*) from " + table + " where GEOMETRY is not null;")
@SuppressWarnings("java:S2077") // table name checked against a regex
var result = stmt.executeQuery("SELECT COUNT(*) FROM %S WHERE GEOMETRY IS NOT NULL;".formatted(table))
) {
count += result.getLong(1);
} catch (SQLException e) {
@ -134,10 +140,11 @@ public class NaturalEarthReader extends SimpleReader {
var tables = tableNames();
for (int i = 0; i < tables.size(); i++) {
String table = tables.get(i);
LOGGER.trace("Naturalearth loading " + i + "/" + tables.size() + ": " + table);
LOGGER.trace("Naturalearth loading {}/{}: {}", i, tables.size(), table);
try (Statement statement = conn.createStatement()) {
ResultSet rs = statement.executeQuery("select * from " + table + ";");
@SuppressWarnings("java:S2077") // table name checked against a regex
ResultSet rs = statement.executeQuery("SELECT * FROM %s;".formatted(table));
String[] column = new String[rs.getMetaData().getColumnCount()];
int geometryColumn = -1;
for (int c = 0; c < column.length; c++) {

Wyświetl plik

@ -37,9 +37,5 @@ public interface OsmBlockSource extends Closeable {
default Iterator<OsmElement> iterator() {
return decodeElements().iterator();
}
default int id() {
return -1;
}
}
}

Wyświetl plik

@ -1,6 +1,7 @@
package com.onthegomap.planetiler.reader.osm;
import com.onthegomap.planetiler.config.Bounds;
import com.onthegomap.planetiler.reader.FileFormatException;
import com.onthegomap.planetiler.util.DiskBacked;
import com.onthegomap.planetiler.util.FileUtils;
import java.io.IOException;
@ -82,7 +83,7 @@ public class OsmInputFile implements Bounds.Provider, Supplier<OsmBlockSource>,
.filter(feature -> !(feature.equals("OsmSchema-V0.6") || feature.equals("DenseNodes")))
.toList();
if (!unsupportedFeatures.isEmpty()) {
throw new RuntimeException("PBF file contains unsupported features " + unsupportedFeatures);
throw new FileFormatException("PBF file contains unsupported features " + unsupportedFeatures);
}
}
@ -146,7 +147,6 @@ public class OsmInputFile implements Bounds.Provider, Supplier<OsmBlockSource>,
@Override
public void forEachBlock(Consumer<Block> consumer) {
int blockId = 0;
try (FileChannel channel = openChannel()) {
final long size = channel.size();
while (channel.position() < size) {
@ -154,11 +154,11 @@ public class OsmInputFile implements Bounds.Provider, Supplier<OsmBlockSource>,
byte[] blockBytes = readBytes(channel, header.getDatasize());
String headerType = header.getType();
if ("OSMData".equals(headerType)) {
consumer.accept(new EagerBlock(blockId++, blockBytes));
consumer.accept(new EagerBlock(blockBytes));
} else if ("OSMHeader".equals(headerType)) {
validateHeader(blockBytes);
} else {
LOGGER.warn("Unrecognized OSM PBF blob header type: " + headerType);
LOGGER.warn("Unrecognized OSM PBF blob header type: {}", headerType);
}
}
} catch (IOException e) {
@ -166,7 +166,13 @@ public class OsmInputFile implements Bounds.Provider, Supplier<OsmBlockSource>,
}
}
private record EagerBlock(@Override int id, byte[] bytes) implements Block {
private static final class EagerBlock implements Block {
// not a record since would need to override equals/hashcode for byte array anyway
private final byte[] bytes;
private EagerBlock(byte[] bytes) {
this.bytes = bytes;
}
public Iterable<OsmElement> decodeElements() {
return PbfDecoder.decode(bytes);
@ -186,7 +192,6 @@ public class OsmInputFile implements Bounds.Provider, Supplier<OsmBlockSource>,
@Override
public void forEachBlock(Consumer<Block> consumer) {
int blockId = 0;
try (FileChannel channel = openChannel()) {
final long size = channel.size();
while (channel.position() < size) {
@ -195,11 +200,11 @@ public class OsmInputFile implements Bounds.Provider, Supplier<OsmBlockSource>,
String headerType = header.getType();
long blockStartPosition = channel.position();
if ("OSMData".equals(headerType)) {
consumer.accept(new LazyBlock(blockId++, blockStartPosition, blockSize, lazyReadChannel));
consumer.accept(new LazyBlock(blockStartPosition, blockSize, lazyReadChannel));
} else if ("OSMHeader".equals(headerType)) {
validateHeader(readBytes(channel, blockStartPosition, blockSize));
} else {
LOGGER.warn("Unrecognized OSM PBF blob header type: " + headerType);
LOGGER.warn("Unrecognized OSM PBF blob header type: {}", headerType);
}
channel.position(blockStartPosition + blockSize);
}
@ -217,7 +222,7 @@ public class OsmInputFile implements Bounds.Provider, Supplier<OsmBlockSource>,
}
}
private record LazyBlock(@Override int id, long offset, int length, FileChannel channel) implements Block {
private record LazyBlock(long offset, int length, FileChannel channel) implements Block {
public Iterable<OsmElement> decodeElements() {
try {

Wyświetl plik

@ -4,6 +4,7 @@ package com.onthegomap.planetiler.reader.osm;
import com.carrotsearch.hppc.LongArrayList;
import com.google.common.collect.Iterators;
import com.onthegomap.planetiler.reader.FileFormatException;
import java.io.IOException;
import java.io.UncheckedIOException;
import java.time.Instant;
@ -13,6 +14,7 @@ import java.util.HashMap;
import java.util.Iterator;
import java.util.List;
import java.util.Map;
import java.util.NoSuchElementException;
import java.util.function.IntUnaryOperator;
import java.util.zip.DataFormatException;
import java.util.zip.Inflater;
@ -50,14 +52,14 @@ public class PbfDecoder implements Iterable<OsmElement> {
try {
inflater.inflate(blobData);
} catch (DataFormatException e) {
throw new RuntimeException("Unable to decompress PBF blob.", e);
throw new FileFormatException("Unable to decompress PBF blob.", e);
}
if (!inflater.finished()) {
throw new RuntimeException("PBF blob contains incomplete compressed data.");
throw new FileFormatException("PBF blob contains incomplete compressed data.");
}
inflater.end();
} else {
throw new RuntimeException("PBF blob uses unsupported compression, only raw or zlib may be used.");
throw new FileFormatException("PBF blob uses unsupported compression, only raw or zlib may be used.");
}
return blobData;
@ -139,6 +141,9 @@ public class PbfDecoder implements Iterable<OsmElement> {
@Override
public OsmElement.Node next() {
if (!hasNext()) {
throw new NoSuchElementException();
}
var node = nodes.get(i++);
return new OsmElement.Node(
node.getId(),
@ -166,23 +171,26 @@ public class PbfDecoder implements Iterable<OsmElement> {
@Override
public OsmElement.Relation next() {
if (!hasNext()) {
throw new NoSuchElementException();
}
var relation = relations.get(i++);
int num = relation.getMemidsCount();
List<OsmElement.Relation.Member> members = new ArrayList<>(num);
long memberId = 0;
for (int i = 0; i < num; i++) {
memberId += relation.getMemids(i);
var memberType = switch (relation.getTypes(i)) {
case WAY -> OsmElement.Relation.Type.WAY;
case NODE -> OsmElement.Relation.Type.NODE;
case RELATION -> OsmElement.Relation.Type.RELATION;
for (int j = 0; j < num; j++) {
memberId += relation.getMemids(j);
var memberType = switch (relation.getTypes(j)) {
case WAY -> OsmElement.Type.WAY;
case NODE -> OsmElement.Type.NODE;
case RELATION -> OsmElement.Type.RELATION;
};
members.add(new OsmElement.Relation.Member(
memberType,
memberId,
fieldDecoder.decodeString(relation.getRolesSid(i))
fieldDecoder.decodeString(relation.getRolesSid(j))
));
}
@ -212,6 +220,9 @@ public class PbfDecoder implements Iterable<OsmElement> {
@Override
public OsmElement.Way next() {
if (!hasNext()) {
throw new NoSuchElementException();
}
var way = ways.get(i++);
// Build up the list of way nodes for the way. The node ids are
// delta encoded meaning that each id is stored as a delta against
@ -221,10 +232,10 @@ public class PbfDecoder implements Iterable<OsmElement> {
LongArrayList wayNodesList = new LongArrayList(numNodes);
wayNodesList.elementsCount = numNodes;
long[] wayNodes = wayNodesList.buffer;
for (int i = 0; i < numNodes; i++) {
long nodeIdOffset = way.getRefs(i);
for (int j = 0; j < numNodes; j++) {
long nodeIdOffset = way.getRefs(j);
nodeId += nodeIdOffset;
wayNodes[i] = nodeId;
wayNodes[j] = nodeId;
}
return new OsmElement.Way(
@ -261,6 +272,9 @@ public class PbfDecoder implements Iterable<OsmElement> {
@Override
public OsmElement.Node next() {
if (!hasNext()) {
throw new NoSuchElementException();
}
// Delta decode node fields.
nodeId += nodes.getId(i);
latitude += nodes.getLat(i);

Wyświetl plik

@ -4,12 +4,13 @@ import com.onthegomap.planetiler.util.Format;
import java.time.Duration;
import java.util.Locale;
import java.util.Optional;
import javax.annotation.concurrent.Immutable;
/**
* A utility for measuring the wall and CPU time that this JVM consumes between snapshots.
* <p>
* For example:
*
*
* <pre>
* {@code
* var start = ProcessTime.now();
@ -19,6 +20,7 @@ import java.util.Optional;
* }
* </pre>
*/
@Immutable
public record ProcessTime(Duration wall, Optional<Duration> cpu, Duration gc) {
/** Takes a snapshot of current wall and CPU time of this JVM. */

Wyświetl plik

@ -1,5 +1,6 @@
package com.onthegomap.planetiler.stats;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import static com.onthegomap.planetiler.util.Format.padLeft;
import static com.onthegomap.planetiler.util.Format.padRight;
@ -100,7 +101,7 @@ public class ProgressLoggers {
long now = System.nanoTime();
long valueNow = getValue.getAsLong();
double timeDiff = (now - lastTime.get()) * 1d / (1d * TimeUnit.SECONDS.toNanos(1));
double valueDiff = valueNow - last.get();
long valueDiff = valueNow - last.get();
if (valueDiff < 0) {
valueDiff = valueNow;
}
@ -160,7 +161,7 @@ public class ProgressLoggers {
long now = System.nanoTime();
long valueNow = getValue.getAsLong();
double timeDiff = (now - lastTime.get()) * 1d / (1d * TimeUnit.SECONDS.toNanos(1));
double valueDiff = valueNow - last.get();
long valueDiff = valueNow - last.get();
if (valueDiff < 0) {
valueDiff = valueNow;
}
@ -321,7 +322,7 @@ public class ProgressLoggers {
lastThreads.putAll(newThreads);
return (first ? " " : " -> ") + name + percents;
}));
} catch (Throwable ignored) {
} catch (Exception ignored) {
// can't get CPU stats per-thread
}
return this;
@ -381,8 +382,11 @@ public class ProgressLoggers {
try {
future.get(duration.toNanos(), TimeUnit.NANOSECONDS);
return true;
} catch (InterruptedException | ExecutionException e) {
throw new IllegalStateException(e);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
return throwFatalException(e);
} catch (ExecutionException e) {
return throwFatalException(e);
} catch (TimeoutException e) {
return false;
}

Wyświetl plik

@ -19,7 +19,7 @@ import org.slf4j.LoggerFactory;
@ThreadSafe
public class Timers {
private static final Logger LOGGER = LoggerFactory.getLogger(Stats.InMemory.class);
private static final Logger LOGGER = LoggerFactory.getLogger(Timers.class);
private static final Format FORMAT = Format.defaultInstance();
private final Map<String, Stage> timers = Collections.synchronizedMap(new LinkedHashMap<>());
private final AtomicReference<Stage> currentStage = new AtomicReference<>();

Wyświetl plik

@ -11,6 +11,7 @@ import java.io.IOException;
import java.io.InputStream;
import java.util.Comparator;
import java.util.List;
import javax.annotation.concurrent.Immutable;
/**
* A utility to download {@code planet.osm.pbf} files from <a href="https://registry.opendata.aws/osm/">AWS Open Data
@ -75,6 +76,7 @@ public class AwsOsm {
}
@JsonIgnoreProperties(ignoreUnknown = true)
@Immutable
record IndexXml(
@JacksonXmlProperty(localName = "Contents")
@JacksonXmlElementWrapper(useWrapping = false) List<ContentXml> contents

Wyświetl plik

@ -80,7 +80,7 @@ public class Downloader {
this.chunkSizeBytes = chunkSizeBytes;
this.config = config;
this.stats = stats;
this.executor = Executors.newSingleThreadExecutor((runnable) -> {
this.executor = Executors.newSingleThreadExecutor(runnable -> {
Thread thread = new Thread(() -> {
LogUtil.setStage("download");
runnable.run();
@ -173,7 +173,10 @@ public class Downloader {
try {
long size = toDownload.metadata.get(10, TimeUnit.SECONDS).size;
loggers.addStorageRatePercentCounter(toDownload.id, size, toDownload::bytesDownloaded, true);
} catch (InterruptedException | ExecutionException | TimeoutException e) {
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throw new IllegalStateException("Error getting size of " + toDownload.url, e);
} catch (ExecutionException | TimeoutException e) {
throw new IllegalStateException("Error getting size of " + toDownload.url, e);
}
}
@ -182,7 +185,7 @@ public class Downloader {
executor.shutdown();
}
CompletableFuture<?> downloadIfNecessary(ResourceToDownload resourceToDownload) {
CompletableFuture<Void> downloadIfNecessary(ResourceToDownload resourceToDownload) {
long existingSize = FileUtils.size(resourceToDownload.output);
return httpHeadFollowRedirects(resourceToDownload.url, 0)
@ -195,12 +198,12 @@ public class Downloader {
})
.thenComposeAsync(metadata -> {
if (metadata.size == existingSize) {
LOGGER.info("Skipping " + resourceToDownload.id + ": " + resourceToDownload.output + " already up-to-date");
LOGGER.info("Skipping {}: {} already up-to-date", resourceToDownload.id, resourceToDownload.output);
return CompletableFuture.completedFuture(null);
} else {
String redirectInfo = metadata.canonicalUrl.equals(resourceToDownload.url) ? "" :
" (redirected to " + metadata.canonicalUrl + ")";
LOGGER.info("Downloading " + resourceToDownload.url + redirectInfo + " to " + resourceToDownload.output);
LOGGER.info("Downloading {}{} to {}", resourceToDownload.url, redirectInfo, resourceToDownload.output);
FileUtils.delete(resourceToDownload.output);
FileUtils.createParentDirectories(resourceToDownload.output);
Path tmpPath = resourceToDownload.tmpPath();
@ -212,16 +215,16 @@ public class Downloader {
.thenCompose(result -> {
try {
Files.move(tmpPath, resourceToDownload.output);
return CompletableFuture.completedFuture(result);
return CompletableFuture.completedFuture(null);
} catch (IOException e) {
return CompletableFuture.failedFuture(e);
return CompletableFuture.<Void>failedFuture(e);
}
})
.whenCompleteAsync((result, error) -> {
if (error != null) {
LOGGER.error("Error downloading " + resourceToDownload.url + " to " + resourceToDownload.output, error);
LOGGER.error("Error downloading {} to {}", resourceToDownload.url, resourceToDownload.output, error);
} else {
LOGGER.info("Finished downloading " + resourceToDownload.url + " to " + resourceToDownload.output);
LOGGER.info("Finished downloading {} to {}", resourceToDownload.url, resourceToDownload.output);
}
FileUtils.delete(tmpPath);
}, executor);

Wyświetl plik

@ -0,0 +1,40 @@
package com.onthegomap.planetiler.util;
import java.io.IOException;
import java.io.UncheckedIOException;
/**
* Exception-handling utilities.
*/
public class Exceptions {
private Exceptions() {}
/**
* Re-throw a caught exception, handling interrupts and wrapping in a {@link FatalPlanetilerException} if checked.
*
* @param exception The original exception
* @param <T> Return type if caller requires it
*/
public static <T> T throwFatalException(Throwable exception) {
if (exception instanceof InterruptedException) {
Thread.currentThread().interrupt();
}
if (exception instanceof RuntimeException runtimeException) {
throw runtimeException;
} else if (exception instanceof IOException ioe) {
throw new UncheckedIOException(ioe);
} else if (exception instanceof Error error) {
throw error;
}
throw new FatalPlanetilerException(exception);
}
/**
* Fatal exception that will result in planetiler exiting early and shutting down.
*/
public static class FatalPlanetilerException extends RuntimeException {
public FatalPlanetilerException(Throwable exception) {
super(exception);
}
}
}

Wyświetl plik

@ -51,8 +51,8 @@ public class FileUtils {
/** Returns the directory usage of all files until {@code path} or 0 if missing/inaccessible. */
public static long directorySize(Path path) {
try {
return Files.walk(path)
try (var walker = Files.walk(path)) {
return walker
.filter(Files::isRegularFile)
.mapToLong(FileUtils::fileSize)
.sum();
@ -77,8 +77,8 @@ public class FileUtils {
/** Deletes all files under a directory and fails silently if it doesn't exist. */
public static void deleteDirectory(Path path) {
try {
Files.walk(path)
try (var walker = Files.walk(path)) {
walker
.sorted(Comparator.reverseOrder())
.forEach(FileUtils::deleteFile);
} catch (NoSuchFileException e) {

Wyświetl plik

@ -12,6 +12,7 @@ import java.util.Map;
import java.util.Set;
import java.util.stream.Collectors;
import java.util.stream.Stream;
import javax.annotation.concurrent.Immutable;
import javax.annotation.concurrent.ThreadSafe;
/**
@ -46,7 +47,7 @@ public class Geofabrik {
return searchIndexForDownloadUrl(searchQuery, index);
}
private synchronized static IndexJson getAndCacheIndex(PlanetilerConfig config) {
private static synchronized IndexJson getAndCacheIndex(PlanetilerConfig config) {
if (index == null) {
try (
InputStream inputStream = Downloader.openStream("https://download.geofabrik.de/index-v1-nogeom.json",
@ -106,5 +107,6 @@ public class Geofabrik {
record FeatureJson(PropertiesJson properties) {}
@Immutable
record IndexJson(List<FeatureJson> features) {}
}

Wyświetl plik

@ -3,6 +3,7 @@ package com.onthegomap.planetiler.util;
import static com.google.common.net.HttpHeaders.ACCEPT;
import static com.google.common.net.HttpHeaders.CONTENT_TYPE;
import static com.google.common.net.HttpHeaders.USER_AGENT;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import com.carrotsearch.hppc.LongHashSet;
import com.carrotsearch.hppc.LongObjectMap;
@ -253,10 +254,13 @@ public class Wikidata {
Timer timer = Timer.start();
LongObjectMap<Map<String, String>> results = queryWikidata(qidsToFetch);
batches.inc();
LOGGER.info("Fetched batch " + batches.get() + " (" + qidsToFetch.size() + " qids) " + timer.stop());
LOGGER.info("Fetched batch {} ({} qids) {}", batches.get(), qidsToFetch.size(), timer.stop());
writeTranslations(results);
} catch (IOException | InterruptedException e) {
throw new RuntimeException(e);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throwFatalException(e);
} catch (IOException e) {
throwFatalException(e);
}
wikidatas.incBy(qidsToFetch.size());
qidsToFetch.clear();

Wyświetl plik

@ -1,5 +1,7 @@
package com.onthegomap.planetiler.worker;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import com.onthegomap.planetiler.collection.IterableOnce;
import java.util.ArrayDeque;
import java.util.Queue;
@ -17,7 +19,7 @@ import java.util.concurrent.TimeUnit;
*/
public class WeightedHandoffQueue<T> implements AutoCloseable, IterableOnce<T> {
private final Queue<T> DONE = new ArrayDeque<>(0);
private final Queue<T> doneSentinel = new ArrayDeque<>(0);
private final BlockingQueue<Queue<T>> itemQueue;
private final int writeLimit;
private boolean done = false;
@ -38,9 +40,12 @@ public class WeightedHandoffQueue<T> implements AutoCloseable, IterableOnce<T> {
public void close() {
try {
flushWrites();
itemQueue.put(DONE);
itemQueue.put(doneSentinel);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throwFatalException(e);
} catch (Exception e) {
throw new RuntimeException(e);
throwFatalException(e);
}
}
@ -64,8 +69,9 @@ public class WeightedHandoffQueue<T> implements AutoCloseable, IterableOnce<T> {
writeBatch = null;
writeCost = 0;
itemQueue.put(oldWriteBatch);
} catch (InterruptedException ex) {
throw new RuntimeException(ex);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throwFatalException(e);
}
}
}
@ -85,7 +91,7 @@ public class WeightedHandoffQueue<T> implements AutoCloseable, IterableOnce<T> {
try {
itemBatch = itemQueue.poll(100, TimeUnit.MILLISECONDS);
if (itemBatch != null) {
if (itemBatch == DONE) {
if (itemBatch == doneSentinel) {
done = true;
}
break;
@ -94,7 +100,7 @@ public class WeightedHandoffQueue<T> implements AutoCloseable, IterableOnce<T> {
Thread.currentThread().interrupt();
break;// signal EOF
}
} else if (itemBatch == DONE) {
} else if (itemBatch == doneSentinel) {
done = true;
}
} while (itemBatch == null);

Wyświetl plik

@ -1,5 +1,7 @@
package com.onthegomap.planetiler.worker;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import com.onthegomap.planetiler.collection.IterableOnce;
import com.onthegomap.planetiler.stats.Counter;
import com.onthegomap.planetiler.stats.Stats;
@ -76,8 +78,11 @@ public class WorkQueue<T> implements AutoCloseable, IterableOnce<T>, Consumer<T>
}
}
hasIncomingData = false;
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throwFatalException(e);
} catch (Exception e) {
throw new RuntimeException(e);
throwFatalException(e);
}
}
@ -169,7 +174,8 @@ public class WorkQueue<T> implements AutoCloseable, IterableOnce<T>, Consumer<T>
enqueueBlockTimeNanos.incBy(System.nanoTime() - start);
}
} catch (InterruptedException ex) {
throw new RuntimeException(ex);
Thread.currentThread().interrupt();
throwFatalException(ex);
}
}
}

Wyświetl plik

@ -1,10 +1,10 @@
package com.onthegomap.planetiler.worker;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import com.onthegomap.planetiler.stats.ProgressLoggers;
import com.onthegomap.planetiler.stats.Stats;
import com.onthegomap.planetiler.util.LogUtil;
import java.io.IOException;
import java.io.UncheckedIOException;
import java.time.Duration;
import java.util.ArrayList;
import java.util.Collection;
@ -25,7 +25,7 @@ public class Worker {
private static final Logger LOGGER = LoggerFactory.getLogger(Worker.class);
private final String prefix;
private final CompletableFuture<?> done;
private final CompletableFuture<Void> done;
private static final AtomicBoolean firstWorkerDied = new AtomicBoolean(false);
/**
@ -36,6 +36,7 @@ public class Worker {
* @param threads number of parallel threads to run {@code task} in
* @param task the work to do in each thread
*/
@SuppressWarnings("java:S1181")
public Worker(String prefix, Stats stats, int threads, RunnableThatThrows task) {
this.prefix = prefix;
stats.gauge(prefix + "_threads", threads);
@ -55,9 +56,9 @@ public class Worker {
System.err.println("Worker " + id + " died");
// when one worker dies it may close resources causing others to die as well, so only log the first
if (firstWorkerDied.compareAndSet(false, true)) {
e.printStackTrace();
e.printStackTrace(); // NOSONAR
}
throwRuntimeException(e);
throwFatalException(e);
} finally {
LOGGER.trace("Finished worker");
}
@ -71,7 +72,7 @@ public class Worker {
* Returns a future that completes successfully when all {@code futures} complete, or fails immediately when the first
* one fails.
*/
public static CompletableFuture<?> joinFutures(CompletableFuture<?>... futures) {
public static CompletableFuture<Void> joinFutures(CompletableFuture<?>... futures) {
return joinFutures(List.of(futures));
}
@ -79,7 +80,7 @@ public class Worker {
* Returns a future that completes successfully when all {@code futures} complete, or fails immediately when the first
* one fails.
*/
public static CompletableFuture<?> joinFutures(Collection<CompletableFuture<?>> futures) {
public static CompletableFuture<Void> joinFutures(Collection<CompletableFuture<?>> futures) {
CompletableFuture<Void> result = CompletableFuture.allOf(futures.toArray(CompletableFuture[]::new));
// fail fast on exceptions
for (CompletableFuture<?> f : futures) {
@ -93,24 +94,11 @@ public class Worker {
return result;
}
private static void throwRuntimeException(Throwable exception) {
if (exception instanceof RuntimeException runtimeException) {
throw runtimeException;
} else if (exception instanceof IOException ioe) {
throw new UncheckedIOException(ioe);
} else if (exception instanceof Error error) {
throw error;
} else if (exception instanceof InterruptedException) {
Thread.currentThread().interrupt();
}
throw new RuntimeException(exception);
}
public String getPrefix() {
return prefix;
}
public CompletableFuture<?> done() {
public CompletableFuture<Void> done() {
return done;
}
@ -132,8 +120,11 @@ public class Worker {
public void await() {
try {
done().get();
} catch (ExecutionException | InterruptedException e) {
throw new RuntimeException(e);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throwFatalException(e);
} catch (ExecutionException e) {
throwFatalException(e);
}
}

Wyświetl plik

@ -1,5 +1,6 @@
package com.onthegomap.planetiler.worker;
import static com.onthegomap.planetiler.util.Exceptions.throwFatalException;
import static com.onthegomap.planetiler.worker.Worker.joinFutures;
import com.onthegomap.planetiler.collection.IterableOnce;
@ -38,7 +39,7 @@ public record WorkerPipeline<T> (
WorkerPipeline<?> previous,
WorkQueue<T> inputQueue,
Worker worker,
CompletableFuture<?> done
CompletableFuture<Void> done
) {
/*
* Empty/Bufferable/Builder are used to provide a fluent API for building a model of the steps to run (and keep
@ -69,8 +70,11 @@ public record WorkerPipeline<T> (
public void await() {
try {
done.get();
} catch (InterruptedException | ExecutionException e) {
throw new RuntimeException(e);
} catch (InterruptedException e) {
Thread.currentThread().interrupt();
throwFatalException(e);
} catch (ExecutionException e) {
throwFatalException(e);
}
}
@ -252,7 +256,8 @@ public record WorkerPipeline<T> (
private WorkerPipeline<?> build() {
var previousPipeline = previous == null || previous.worker == null ? null : previous.build();
var doneFuture = worker != null ? worker.done() : CompletableFuture.completedFuture(true);
CompletableFuture<Void> doneFuture =
worker != null ? worker.done() : CompletableFuture.completedFuture(null);
if (previousPipeline != null) {
doneFuture = joinFutures(doneFuture, previousPipeline.done);
}

Wyświetl plik

@ -3,6 +3,7 @@
package vector_tile;
@javax.annotation.Generated(value = "protoc", comments = "annotations:VectorTileProto.java.pb.meta")
public final class VectorTileProto {
private VectorTileProto() {}

Wyświetl plik

@ -1,11 +1,15 @@
package com.onthegomap.planetiler.collection;
import static org.junit.jupiter.api.Assertions.*;
import static org.junit.jupiter.api.Assertions.assertEquals;
import static org.junit.jupiter.api.Assertions.assertFalse;
import static org.junit.jupiter.api.Assertions.assertThrows;
import static org.junit.jupiter.api.Assertions.assertTrue;
import java.util.ArrayList;
import java.util.HashSet;
import java.util.LinkedList;
import java.util.List;
import java.util.NoSuchElementException;
import java.util.Queue;
import java.util.Set;
import java.util.stream.Stream;
@ -18,9 +22,9 @@ public class IterableOnceTest {
IterableOnce<Integer> empty = () -> null;
var iter = empty.iterator();
assertFalse(iter.hasNext());
assertNull(iter.next());
assertThrows(NoSuchElementException.class, iter::next);
assertFalse(iter.hasNext());
assertNull(iter.next());
assertThrows(NoSuchElementException.class, iter::next);
}
@Test
@ -31,7 +35,7 @@ public class IterableOnceTest {
assertTrue(iter.hasNext());
assertEquals(1, iter.next());
assertFalse(iter.hasNext());
assertNull(iter.next());
assertThrows(NoSuchElementException.class, iter::next);
}
@Test
@ -44,7 +48,7 @@ public class IterableOnceTest {
assertTrue(iter.hasNext());
assertEquals(2, iter.next());
assertFalse(iter.hasNext());
assertNull(iter.next());
assertThrows(NoSuchElementException.class, iter::next);
}
@Test

Wyświetl plik

@ -26,6 +26,7 @@
<sonar.organization>onthegomap</sonar.organization>
<sonar.projectKey>onthegomap_planetiler</sonar.projectKey>
<sonar.moduleKey>${project.artifactId}</sonar.moduleKey>
<sonar.exclusions>planetiler-benchmarks/**/*</sonar.exclusions>
</properties>
<scm>

Wyświetl plik

@ -2,7 +2,7 @@
set -ex
echo "Regenerating..."
protoc --java_out=planetiler-core/src/main/java/ planetiler-core/src/main/resources/vector_tile_proto.proto
protoc --java_out=annotate_code:planetiler-core/src/main/java/ planetiler-core/src/main/resources/vector_tile_proto.proto
echo "Formatting..."
./scripts/format.sh

Wyświetl plik

@ -1 +1,14 @@
sonar.exclusions=**/VectorTileProto.java
sonar.issue.ignore.multicriteria=java_S1659,java_S3358,java_S1172,java_S106,java_S125
# subjective
sonar.issue.ignore.multicriteria.java_S1659.ruleKey=java:S1659
sonar.issue.ignore.multicriteria.java_S1659.resourceKey=**/*.java
sonar.issue.ignore.multicriteria.java_S3358.ruleKey=java:S3358
sonar.issue.ignore.multicriteria.java_S3358.resourceKey=**/*.java
sonar.issue.ignore.multicriteria.java_SS106.ruleKey=java:S106
sonar.issue.ignore.multicriteria.java_SS106.resourceKey=**/*.java
sonar.issue.ignore.multicriteria.java_SS125.ruleKey=java:S125
sonar.issue.ignore.multicriteria.java_SS125.resourceKey=**/*.java
# layer constructors need same signatures
sonar.issue.ignore.multicriteria.java_S1172.ruleKey=java:S1172
sonar.issue.ignore.multicriteria.java_S1172.resourceKey=**/basemap/layers/*.java