logging improvements

pull/1/head
Mike Barry 2021-08-10 06:55:30 -04:00
rodzic 4af6d1707b
commit 994bdb69eb
25 zmienionych plików z 274 dodań i 101 usunięć

1
.gitignore vendored
Wyświetl plik

@ -2,6 +2,7 @@
target/
*.jar
*.log
.idea/*
*.iml

Wyświetl plik

@ -13,6 +13,7 @@ import com.onthegomap.flatmap.reader.osm.OsmInputFile;
import com.onthegomap.flatmap.stats.Stats;
import com.onthegomap.flatmap.stats.Timers;
import com.onthegomap.flatmap.util.FileUtils;
import com.onthegomap.flatmap.util.LogUtil;
import com.onthegomap.flatmap.worker.Worker;
import java.nio.file.Files;
import java.nio.file.Path;
@ -48,7 +49,8 @@ public class FlatMapRunner {
private FlatMapRunner(Arguments arguments) {
this.arguments = arguments;
stats = arguments.getStats();
overallTimer = stats.startTimer("overall");
overallTimer = stats.startStage("overall");
LogUtil.clearStage();
tmpDir = arguments.file("tmpdir", "temp directory", Path.of("data", "tmp"));
}
@ -101,7 +103,9 @@ public class FlatMapRunner {
if (profile.caresAboutSource(name)) {
task.run();
} else {
LOGGER.info("[" + name + "] skipping since profile does not use it");
LogUtil.setStage(name);
LOGGER.info("Skipping since profile does not use it");
LogUtil.clearStage();
}
};
}
@ -177,11 +181,11 @@ public class FlatMapRunner {
for (String stage : stages.keySet()) {
for (String substage : subStages.getOrDefault(stage, List.of(stage))) {
LOGGER.info(" [" + substage + "] " + stageDescriptions.getOrDefault(substage, ""));
LOGGER.info(" " + substage + ": " + stageDescriptions.getOrDefault(substage, ""));
}
}
LOGGER.info(" [sort] Sort rendered features by tile ID");
LOGGER.info(" [mbtiles] Encode each tile and write to " + output);
LOGGER.info(" sort: Sort rendered features by tile ID");
LOGGER.info(" mbtiles: Encode each tile and write to " + output);
Files.createDirectories(tmpDir);
nodeDbPath = tmpDir.resolve("node.db");

Wyświetl plik

@ -126,13 +126,12 @@ public class Wikidata {
}
public static void fetch(OsmInputFile infile, Path outfile, CommonParams config, Profile profile, Stats stats) {
int threads = config.threads();
var timer = stats.startTimer("wikidata");
var timer = stats.startStage("wikidata");
int threadsAvailable = Math.max(1, config.threads() - 2);
int processThreads = Math.max(1, threadsAvailable / 4);
int readerThreads = Math.max(1, threadsAvailable - processThreads);
LOGGER
.info("[wikidata] Starting with " + readerThreads + " reader threads and " + processThreads + " process threads");
.info("Starting with " + readerThreads + " reader threads and " + processThreads + " process threads");
WikidataTranslations oldMappings = load(outfile);
try (Writer writer = Files.newBufferedWriter(outfile)) {
@ -155,17 +154,19 @@ public class Wikidata {
});
ProgressLoggers loggers = new ProgressLoggers("wikidata")
.addRateCounter("nodes", fetcher.nodes)
.addRateCounter("ways", fetcher.ways)
.addRateCounter("rels", fetcher.rels)
.addRateCounter("nodes", fetcher.nodes, true)
.addRateCounter("ways", fetcher.ways, true)
.addRateCounter("rels", fetcher.rels, true)
.addRateCounter("wiki", fetcher.wikidatas)
.addFileSize(outfile)
.newLine()
.addProcessStats()
.newLine()
.addThreadPoolStats("parse", pbfParsePrefix + "-pool")
.addPipelineStats(pipeline);
pipeline.awaitAndLog(loggers, config.logInterval());
LOGGER.info("[wikidata] DONE fetched:" + fetcher.wikidatas.get());
LOGGER.info("DONE fetched:" + fetcher.wikidatas.get());
} catch (IOException e) {
throw new RuntimeException(e);
}
@ -178,11 +179,11 @@ public class Wikidata {
try (BufferedReader fis = Files.newBufferedReader(path)) {
WikidataTranslations result = load(fis);
LOGGER.info(
"[wikidata] loaded from " + result.getAll().size() + " mappings from " + path.toAbsolutePath() + " in " + watch
"loaded from " + result.getAll().size() + " mappings from " + path.toAbsolutePath() + " in " + watch
.stop());
return result;
} catch (IOException e) {
LOGGER.info("[wikidata] error loading " + path.toAbsolutePath() + ": " + e);
LOGGER.info("error loading " + path.toAbsolutePath() + ": " + e);
return new WikidataTranslations();
}
}
@ -294,7 +295,7 @@ public class Wikidata {
public void loadExisting(WikidataTranslations oldMappings) throws IOException {
LongObjectMap<Map<String, String>> alreadyHave = oldMappings.getAll();
if (!alreadyHave.isEmpty()) {
LOGGER.info("[wikidata] skipping " + alreadyHave.size() + " mappings we already have");
LOGGER.info("skipping " + alreadyHave.size() + " mappings we already have");
writeTranslations(alreadyHave);
for (LongObjectCursor<Map<String, String>> cursor : alreadyHave) {
visited.add(cursor.key);

Wyświetl plik

@ -139,7 +139,7 @@ class ExternalMergeSort implements FeatureSort {
// ok
}
}
var timer = stats.startTimer("sort");
var timer = stats.startStage("sort");
AtomicLong reading = new AtomicLong(0);
AtomicLong writing = new AtomicLong(0);
AtomicLong sorting = new AtomicLong(0);
@ -157,7 +157,9 @@ class ExternalMergeSort implements FeatureSort {
ProgressLoggers loggers = new ProgressLoggers("sort")
.addPercentCounter("chunks", chunks.size(), doneCounter)
.addFileSize(this::getStorageSize)
.newLine()
.addProcessStats()
.newLine()
.addPipelineStats(pipeline);
pipeline.awaitAndLog(loggers, config.logInterval());

Wyświetl plik

@ -73,6 +73,7 @@ public interface LongLongMap extends Closeable {
LocalCounter counter = new LocalCounter();
ProgressLoggers loggers = new ProgressLoggers("write")
.addRatePercentCounter("entries", entries, () -> counter.count)
.newLine()
.addProcessStats();
AtomicReference<String> writeRate = new AtomicReference<>();
new Worker("writer", Stats.inMemory(), 1, () -> {
@ -93,6 +94,7 @@ public interface LongLongMap extends Closeable {
Counter.Readable readCount = Counter.newMultiThreadCounter();
loggers = new ProgressLoggers("read")
.addRateCounter("entries", readCount)
.newLine()
.addProcessStats();
CountDownLatch latch = new CountDownLatch(readers);
for (int i = 0; i < readers; i++) {

Wyświetl plik

@ -70,11 +70,13 @@ public interface LongLongMultimap extends MemoryEstimator.HasEstimate {
private void doPrepare() {
StopWatch watch = new StopWatch().start();
LOGGER.debug("Sorting long long multimap keys...");
LOGGER.debug("Sorting long long multimap...");
long[] sortedKeys = keys.toArray();
// this happens in a worker thread, but it's OK to use parallel sort because
// all other threads will block while we prepare the multimap.
Arrays.parallelSort(sortedKeys);
LOGGER.debug("Sorted keys, now values...");
long[] sortedValues = new long[sortedKeys.length];
int from = 0;
while (from < keys.size()) {

Wyświetl plik

@ -55,9 +55,8 @@ public class MbtilesWriter {
private final LongAccumulator[] maxTileSizesByZoom;
private final FeatureGroup features;
private final AtomicReference<TileCoord> lastTileWritten = new AtomicReference<>();
private final AtomicLong batchLengths = new AtomicLong(0);
private final AtomicLong batches = new AtomicLong(0);
private final AtomicLong maxInputFeaturesPerTile = new AtomicLong(0);
private final AtomicLong maxBatchLength = new AtomicLong(0);
private final AtomicLong minBatchLength = new AtomicLong(Integer.MAX_VALUE);
MbtilesWriter(FeatureGroup features, Mbtiles db, CommonParams config, Profile profile, Stats stats,
LayerStats layerStats) {
@ -93,7 +92,7 @@ public class MbtilesWriter {
public static void writeOutput(FeatureGroup features, Mbtiles output, LongSupplier fileSize, Profile profile,
CommonParams config, Stats stats) {
var timer = stats.startTimer("mbtiles");
var timer = stats.startStage("mbtiles");
MbtilesWriter writer = new MbtilesWriter(features, output, config, profile, stats,
features.layerStats());
@ -132,22 +131,30 @@ public class MbtilesWriter {
.addRateCounter("tiles", writer::tilesEmitted)
.addFileSize(fileSize)
.add(" features ").addFileSize(features::getStorageSize)
.newLine()
.addProcessStats()
.newLine()
.addPipelineStats(encodeBranch)
.addPipelineStats(writeBranch)
.newLine()
.add(string(() -> {
TileCoord lastTile = writer.lastTileWritten.get();
String blurb;
long batches = writer.batches.getAndSet(0);
long batchLen = writer.batchLengths.getAndSet(0);
String avgBatch = batches == 0 ? "-" : Long.toString(batchLen / batches);
long minBatch = writer.minBatchLength.getAndSet(Integer.MAX_VALUE);
long maxBatch = writer.maxBatchLength.getAndSet(0);
String batchSizeRange = (minBatch > 0 && maxBatch < Integer.MAX_VALUE) ? (minBatch + "-" + maxBatch) : "-";
if (lastTile == null) {
blurb = "n/a";
} else {
blurb =
lastTile.z() + "/" + lastTile.x() + "/" + lastTile.y() + " bsize:" + avgBatch + " max: "
+ writer.maxInputFeaturesPerTile.getAndSet(0) + " " + lastTile.getDebugUrl();
var extentForZoom = config.extents().getForZoom(lastTile.z());
int zMinX = extentForZoom.minX();
int zMaxX = extentForZoom.maxX();
blurb = "%d/%d/%d (z%d %s%%) batch sizes: %s %s".formatted(
lastTile.z(), lastTile.x(), lastTile.y(),
lastTile.z(), (100 * (lastTile.x() + 1 - zMinX)) / (zMaxX - zMinX),
batchSizeRange,
lastTile.getDebugUrl()
);
}
return "last tile: " + blurb;
}));
@ -190,15 +197,10 @@ public class MbtilesWriter {
for (var feature : features) {
int z = feature.coord().z();
if (z > currentZoom) {
LOGGER.info("[mbtiles] Starting z" + z);
LOGGER.info("Starting z" + z);
currentZoom = z;
}
long thisTileFeatures = feature.getNumFeaturesToEmit();
maxInputFeaturesPerTile.accumulateAndGet(thisTileFeatures, Long::max);
// TODO move to profile layer cost function
if (z != 13) {
thisTileFeatures /= 10;
}
if (tilesInThisBatch > 0 &&
(tilesInThisBatch >= MAX_TILES_PER_BATCH ||
((featuresInThisBatch + thisTileFeatures) > MAX_FEATURES_PER_BATCH))) {
@ -295,8 +297,8 @@ public class MbtilesWriter {
tilesByZoom[z].inc();
batchSize++;
}
batches.incrementAndGet();
batchLengths.addAndGet(batchSize);
maxBatchLength.accumulateAndGet(batchSize, Long::max);
minBatchLength.accumulateAndGet(batchSize, Long::min);
lastTileWritten.set(lastTile);
}
}

Wyświetl plik

@ -6,6 +6,7 @@ import com.onthegomap.flatmap.config.CommonParams;
import com.onthegomap.flatmap.geo.GeoUtils;
import com.onthegomap.flatmap.stats.Stats;
import com.onthegomap.flatmap.util.FileUtils;
import com.onthegomap.flatmap.util.LogUtil;
import com.onthegomap.flatmap.worker.WorkerPipeline;
import java.io.IOException;
import java.nio.file.FileSystems;
@ -39,6 +40,7 @@ public class NaturalEarthReader extends Reader {
public NaturalEarthReader(String sourceName, Path input, Path tmpDir, Profile profile, Stats stats) {
super(profile, stats, sourceName);
LogUtil.setStage(sourceName);
try {
conn = open(input, tmpDir);
} catch (IOException | SQLException e) {

Wyświetl plik

@ -28,7 +28,7 @@ public abstract class Reader implements Closeable {
}
public final void process(FeatureGroup writer, CommonParams config) {
var timer = stats.startTimer(sourceName);
var timer = stats.startStage(sourceName);
long featureCount = getCount();
int threads = config.threads();
Envelope latLonBounds = config.latLonBounds();
@ -53,6 +53,7 @@ public abstract class Reader implements Closeable {
}
}
})
// output large batches since each input may map to many tiny output features (ie. slicing ocean tiles)
.addBuffer("write_queue", 50_000, 1_000)
.sinkToConsumer("write", 1, (item) -> {
featuresWritten.incrementAndGet();
@ -63,7 +64,9 @@ public abstract class Reader implements Closeable {
.addRatePercentCounter("read", featureCount, featuresRead)
.addRateCounter("write", featuresWritten)
.addFileSize(writer::getStorageSize)
.newLine()
.addProcessStats()
.newLine()
.addPipelineStats(pipeline);
pipeline.awaitAndLog(loggers, config.logInterval());

Wyświetl plik

@ -95,7 +95,7 @@ public class OpenStreetMapReader implements Closeable, MemoryEstimator.HasEstima
}
public void pass1(CommonParams config) {
var timer = stats.startTimer("osm_pass1");
var timer = stats.startStage("osm_pass1");
String pbfParsePrefix = "pbfpass1";
int parseThreads = Math.max(1, config.threads() - 2);
var pipeline = WorkerPipeline.start("osm_pass1", stats)
@ -104,13 +104,15 @@ public class OpenStreetMapReader implements Closeable, MemoryEstimator.HasEstima
.sinkToConsumer("process", 1, this::processPass1);
var loggers = new ProgressLoggers("osm_pass1")
.addRateCounter("nodes", PASS1_NODES)
.addRateCounter("nodes", PASS1_NODES, true)
.addFileSize(nodeDb::fileSize)
.addRateCounter("ways", PASS1_WAYS)
.addRateCounter("rels", PASS1_RELATIONS)
.addRateCounter("ways", PASS1_WAYS, true)
.addRateCounter("rels", PASS1_RELATIONS, true)
.newLine()
.addProcessStats()
.addInMemoryObject("hppc", this)
.addThreadPoolStats("parse", pbfParsePrefix + "-pool")
.newLine()
.addPipelineStats(pipeline);
pipeline.awaitAndLog(loggers, config.logInterval());
timer.stop();
@ -151,7 +153,7 @@ public class OpenStreetMapReader implements Closeable, MemoryEstimator.HasEstima
}
public void pass2(FeatureGroup writer, CommonParams config) {
var timer = stats.startTimer("osm_pass2");
var timer = stats.startStage("osm_pass2");
int readerThreads = Math.max(config.threads() / 4, 1);
int processThreads = config.threads() - 1;
Counter.MultiThreadCounter nodesProcessed = Counter.newMultiThreadCounter();
@ -216,8 +218,10 @@ public class OpenStreetMapReader implements Closeable, MemoryEstimator.HasEstima
.addRatePercentCounter("rels", PASS1_RELATIONS.get(), relsProcessed)
.addRateCounter("features", () -> writer.sorter().size())
.addFileSize(writer::getStorageSize)
.newLine()
.addProcessStats()
.addInMemoryObject("hppc", this)
.newLine()
.addThreadPoolStats("parse", parseThreadPrefix + "-pool")
.addPipelineStats(pipeline);

Wyświetl plik

@ -32,10 +32,36 @@ import org.slf4j.LoggerFactory;
public class ProgressLoggers {
public static final String ANSI_RESET = "\u001B[0m";
public static final String ANSI_RED = "\u001B[31m";
public static final String ANSI_GREEN = "\u001B[32m";
public static final String ANSI_YELLOW = "\u001B[33m";
private static final String COLOR_RESET = "\u001B[0m";
private static final String FG_RED = "\u001B[31m";
private static final String FG_GREEN = "\u001B[32m";
private static final String FG_YELLOW = "\u001B[33m";
private static final String FG_BLUE = "\u001B[34m";
private static final String FG_CYAN = "\u001B[36m";
private static String fg(String fg, String string) {
return fg + string + COLOR_RESET;
}
private static String red(String string) {
return fg(FG_RED, string);
}
private static String green(String string) {
return fg(FG_GREEN, string);
}
private static String yellow(String string) {
return fg(FG_YELLOW, string);
}
private static String blue(String string) {
return fg(FG_BLUE, string);
}
private static String cyan(String string) {
return fg(FG_CYAN, string);
}
private static final Logger LOGGER = LoggerFactory.getLogger(ProgressLoggers.class);
private final List<Object> loggers;
@ -47,12 +73,21 @@ public class ProgressLoggers {
}
public String getLog() {
return "[" + prefix + "]" + loggers.stream()
return loggers.stream()
.map(Object::toString)
.collect(Collectors.joining(""));
.collect(Collectors.joining(""))
.replaceAll("\n\\s*", "\n ");
}
public ProgressLoggers addRateCounter(String name, LongSupplier getValue) {
return addRateCounter(name, getValue, false);
}
public ProgressLoggers addRateCounter(String name, AtomicLong getValue) {
return addRateCounter(name, getValue, false);
}
public ProgressLoggers addRateCounter(String name, LongSupplier getValue, boolean color) {
AtomicLong last = new AtomicLong(getValue.getAsLong());
AtomicLong lastTime = new AtomicLong(System.nanoTime());
loggers.add(new ProgressLogger(name, () -> {
@ -65,14 +100,14 @@ public class ProgressLoggers {
}
last.set(valueNow);
lastTime.set(now);
return ANSI_GREEN + "[ " + formatNumeric(valueNow, true) + " " + formatNumeric(valueDiff / timeDiff, true)
+ "/s ]" + ANSI_RESET;
String result = "[ " + formatNumeric(valueNow, true) + " " + formatNumeric(valueDiff / timeDiff, true) + "/s ]";
return color && valueDiff > 0 ? green(result) : result;
}));
return this;
}
public ProgressLoggers addRateCounter(String name, AtomicLong value) {
return addRateCounter(name, value::get);
public ProgressLoggers addRateCounter(String name, AtomicLong value, boolean color) {
return addRateCounter(name, value::get, color);
}
public ProgressLoggers addRatePercentCounter(String name, long total, AtomicLong value) {
@ -101,8 +136,9 @@ public class ProgressLoggers {
}
last.set(valueNow);
lastTime.set(now);
return ANSI_GREEN + "[ " + format.apply(valueNow, true) + " " + padLeft(formatPercent(1f * valueNow / total), 4)
+ " " + format.apply(valueDiff / timeDiff, true) + "/s ]" + ANSI_RESET;
String result = "[ " + format.apply(valueNow, true) + " " + padLeft(formatPercent(1f * valueNow / total), 4)
+ " " + format.apply(valueDiff / timeDiff, true) + "/s ]";
return valueDiff > 0 ? green(result) : result;
}));
return this;
}
@ -110,8 +146,8 @@ public class ProgressLoggers {
public ProgressLoggers addPercentCounter(String name, long total, AtomicLong getValue) {
loggers.add(new ProgressLogger(name, () -> {
long valueNow = getValue.get();
return ANSI_GREEN + "[ " + padLeft("" + valueNow, 3) + " / " + padLeft("" + total, 3) + " " + padLeft(
formatPercent(1f * valueNow / total), 4) + " ]" + ANSI_RESET;
return "[ " + padLeft("" + valueNow, 3) + " / " + padLeft("" + total, 3) + " " + padLeft(
formatPercent(1f * valueNow / total), 4) + " ]";
}));
return this;
}
@ -160,15 +196,17 @@ public class ProgressLoggers {
}
public ProgressLoggers addProcessStats() {
add("\n" + " ".repeat(3));
addOptionalDeltaLogger("cpus", ProcessInfo::getProcessCpuTime, Format::formatDecimal);
addDeltaLogger("gc", ProcessInfo::getGcTime, Format::formatPercent);
addOptionalDeltaLogger("cpus", ProcessInfo::getProcessCpuTime, num -> blue(Format.formatDecimal(num)));
addDeltaLogger("gc", ProcessInfo::getGcTime, num -> {
String formatted = Format.formatPercent(num);
return num > 0.3 ? yellow(formatted) : num > 0.6 ? red(formatted) : formatted;
});
loggers.add(new ProgressLogger("mem",
() ->
formatMB(Helper.getUsedMB(), false) + " / " +
formatMB(Helper.getUsedMB(), false) + "/" +
formatMB(Helper.getTotalMB(), false) +
ProcessInfo.getMemoryUsageAfterLastGC().stream()
.mapToObj(value -> " postGC: " + formatBytes(value, false))
.mapToObj(value -> " postGC: " + blue(formatBytes(value, false)))
.findFirst()
.orElse("")
));
@ -220,9 +258,9 @@ public class ProgressLoggers {
lastTime.set(currentTime);
lastThreads.putAll(newThreads);
return (first ? "\n " : " -> ") + name + percents;
return (first ? " " : " -> ") + name + percents;
}));
} catch (Throwable e) {
} catch (Throwable ignored) {
// can't get CPU stats per-thread
}
return this;
@ -263,7 +301,7 @@ public class ProgressLoggers {
}
public ProgressLoggers newLine() {
return add("\n ");
return add("\n");
}
public void awaitAndLog(Future<?> future, Duration logInterval) {

Wyświetl plik

@ -89,11 +89,6 @@ public class PrometheusStats implements Stats {
}
}
@Override
public Timers.Finishable startTimer(String name) {
return timers.startTimer(name);
}
@Override
public void gauge(String name, Supplier<Number> value) {
new Collector() {

Wyświetl plik

@ -2,6 +2,7 @@ package com.onthegomap.flatmap.stats;
import static io.prometheus.client.Collector.NANOSECONDS_PER_SECOND;
import com.onthegomap.flatmap.util.LogUtil;
import com.onthegomap.flatmap.util.MemoryEstimator;
import java.nio.file.Path;
import java.time.Duration;
@ -14,7 +15,14 @@ public interface Stats extends AutoCloseable {
timers().printSummary();
}
Timers.Finishable startTimer(String name);
default Timers.Finishable startStage(String name) {
LogUtil.setStage(name);
var timer = timers().startTimer(name);
return () -> {
timer.stop();
LogUtil.clearStage();
};
}
default void gauge(String name, Number value) {
gauge(name, () -> value);
@ -64,11 +72,6 @@ public interface Stats extends AutoCloseable {
private final Timers timers = new Timers();
@Override
public Timers.Finishable startTimer(String name) {
return timers.startTimer(name);
}
@Override
public void wroteTile(int zoom, int bytes) {
}

Wyświetl plik

@ -23,8 +23,8 @@ public class Timers {
public Finishable startTimer(String name) {
Timer timer = new Timer().start();
timers.put(name, timer);
LOGGER.info("[" + name + "] Starting...");
return () -> LOGGER.info("[" + name + "] Finished in " + timers.get(name).stop() + "\n");
LOGGER.info("Starting...");
return () -> LOGGER.info("Finished in " + timers.get(name).stop() + "\n");
}
public Map<String, Timer> all() {

Wyświetl plik

@ -0,0 +1,31 @@
package com.onthegomap.flatmap.util;
import java.time.Duration;
import org.apache.logging.log4j.core.LogEvent;
import org.apache.logging.log4j.core.config.plugins.Plugin;
import org.apache.logging.log4j.core.lookup.StrLookup;
/**
* A log4j plugin that substitutes {@code $${uptime:now}} pattern with the elapsed time of the program in H:MM:SS form.
*/
@Plugin(name = "uptime", category = StrLookup.CATEGORY)
public class ElapsedTimeLog4jLookup implements StrLookup {
private static final long startTime = System.nanoTime();
@Override
public String lookup(String key) {
Duration duration = Duration.ofNanos(System.nanoTime() - startTime);
return "%d:%02d:%02d".formatted(
duration.toHours(),
duration.toMinutesPart(),
duration.toSecondsPart()
);
}
@Override
public String lookup(LogEvent event, String key) {
return lookup(key);
}
}

Wyświetl plik

@ -0,0 +1,27 @@
package com.onthegomap.flatmap.util;
import java.util.regex.Pattern;
import org.slf4j.MDC;
public class LogUtil {
public static void setStage(String stage) {
MDC.put("stage", stage);
}
public static void clearStage() {
MDC.remove("stage");
}
public static String getStage() {
return MDC.get("stage");
}
public static void setStage(String parent, String childThread) {
if (parent == null) {
setStage(childThread);
} else {
setStage(parent + ":" + childThread.replaceFirst("^" + Pattern.quote(parent) + "_?", ""));
}
}
}

Wyświetl plik

@ -35,6 +35,20 @@ public class Parse {
}
}
private static final Pattern TO_ROUND_INT_SUBSTRING_PATTERN = Pattern.compile("^(-?[\\d.]+)(\\D|$)");
public static Integer parseRoundInt(String tag) {
if (tag == null) {
return null;
}
try {
var matcher = TO_ROUND_INT_SUBSTRING_PATTERN.matcher(tag);
return matcher.find() ? Math.round(Float.parseFloat(matcher.group(1))) : null;
} catch (NumberFormatException e) {
return null;
}
}
public static Integer parseIntOrNull(Object tag) {
if (tag instanceof Number num) {
return num.intValue();

Wyświetl plik

@ -2,6 +2,7 @@ package com.onthegomap.flatmap.worker;
import com.onthegomap.flatmap.stats.ProgressLoggers;
import com.onthegomap.flatmap.stats.Stats;
import com.onthegomap.flatmap.util.LogUtil;
import java.io.IOException;
import java.io.UncheckedIOException;
import java.time.Duration;
@ -55,9 +56,11 @@ public class Worker {
this.prefix = prefix;
stats.gauge(prefix + "_threads", threads);
var es = Executors.newFixedThreadPool(threads, new NamedThreadFactory(prefix));
String parentStage = LogUtil.getStage();
List<CompletableFuture<?>> results = new ArrayList<>();
for (int i = 0; i < threads; i++) {
results.add(CompletableFuture.runAsync(() -> {
LogUtil.setStage(parentStage, prefix);
String id = Thread.currentThread().getName();
LOGGER.trace("Starting worker");
try {

Wyświetl plik

@ -2,7 +2,8 @@ appenders=console
appender.console.type=Console
appender.console.name=STDOUT
appender.console.layout.type=PatternLayout
appender.console.layout.pattern=%d{HH:mm:ss.SSS} [%t] %.-3p - %m%n
appender.console.layout.pattern=%highlight{$${uptime:now} %level{length=3} %notEmpty{[%X{stage}] }- %msg%n%throwable}{FATAL=red, ERROR=red, WARN=YELLOW, INFO=normal, DEBUG=normal, TRACE=normal}
packages=com.onthegomap.flatmap.util
rootLogger.level=debug
rootLogger.appenderRefs=stdout
rootLogger.appenderRef.stdout.ref=STDOUT

Wyświetl plik

@ -32,17 +32,18 @@ public class ProgressLoggersTest {
});
var loggers = new ProgressLoggers("prefix")
.newLine()
.addPipelineStats(pipeline);
readyLatch.await();
String log = loggers.getLog();
assertEquals("[prefix]\n reader( 0%) -> (0/13) -> worker( 0% 0%) -> (0/14) -> writer( 0% 0%)",
assertEquals("\n reader( 0%) -> (0/13) -> worker( 0% 0%) -> (0/14) -> writer( 0% 0%)",
log.replaceAll("[ 0-9][0-9]%", " 0%"));
continueLatch.countDown();
pipeline.awaitAndLog(loggers, Duration.ofSeconds(10));
loggers.getLog();
assertEquals("[prefix]\n reader( -%) -> (0/13) -> worker( -% -%) -> (0/14) -> writer( -% -%)",
assertEquals("\n reader( -%) -> (0/13) -> worker( -% -%) -> (0/14) -> writer( -% -%)",
loggers.getLog());
}
}

Wyświetl plik

@ -34,7 +34,7 @@ public class PrometheusStatsTest {
@Test
public void testTimer() {
PrometheusStats stats = new PrometheusStats("job");
var timer = stats.startTimer("task1");
var timer = stats.startStage("task1");
assertContainsStat("^flatmap_task1_running 1", stats);
assertContainsStat("^flatmap_task1_elapsed_time_seconds [0-9\\.]+$", stats);
assertContainsStat("^flatmap_task1_cpu_time_seconds [0-9\\.]+$", stats);

Wyświetl plik

@ -2,7 +2,8 @@ appenders=console
appender.console.type=Console
appender.console.name=STDOUT
appender.console.layout.type=PatternLayout
appender.console.layout.pattern=%d{HH:mm:ss.SSS} [%t] %.-3p - %m%n
appender.console.layout.pattern=$${uptime:now} %level{length=3} %notEmpty{[%X{stage}] }- %msg%n%throwable
packages=com.onthegomap.flatmap.util
rootLogger.level=warn
rootLogger.appenderRefs=stdout
rootLogger.appenderRef.stdout.ref=STDOUT

Wyświetl plik

@ -65,7 +65,9 @@ import java.util.Map;
import java.util.Set;
import java.util.function.Consumer;
import java.util.function.Function;
import java.util.stream.Collectors;
import org.jetbrains.annotations.NotNull;
import org.locationtech.jts.geom.Coordinate;
import org.locationtech.jts.geom.Geometry;
import org.locationtech.jts.geom.LineString;
import org.locationtech.jts.geom.Point;
@ -89,7 +91,7 @@ public class Boundary implements
OpenMapTilesProfile.FinishHandler {
private static final Logger LOGGER = LoggerFactory.getLogger(Boundary.class);
private static final double COUNTRY_TEST_OFFSET = GeoUtils.metersToPixelAtEquator(0, 100) / 256d;
private static final double COUNTRY_TEST_OFFSET = GeoUtils.metersToPixelAtEquator(0, 10) / 256d;
private final Map<Long, String> regionNames = new HashMap<>();
private final Map<Long, List<Geometry>> regionGeometries = new HashMap<>();
private final Map<CountryBoundaryComponent, List<Geometry>> boundariesToMerge = new HashMap<>();
@ -156,7 +158,7 @@ public class Boundary implements
if (relation.hasTag("type", "boundary") &&
relation.hasTag("admin_level") &&
relation.hasTag("boundary", "administrative")) {
Integer adminLevelValue = Parse.parseIntSubstring(relation.getTag("admin_level"));
Integer adminLevelValue = Parse.parseRoundInt(relation.getTag("admin_level"));
String code = relation.getTag("ISO3166-1:alpha3");
if (adminLevelValue != null && adminLevelValue >= 2 && adminLevelValue <= 10) {
boolean disputed = isDisputed(ReaderElementUtils.getProperties(relation));
@ -262,7 +264,7 @@ public class Boundary implements
public void finish(String sourceName, FeatureCollector.Factory featureCollectors,
Consumer<FeatureCollector.Feature> next) {
if (OpenMapTilesProfile.OSM_SOURCE.equals(sourceName)) {
var timer = stats.startTimer("boundaries");
var timer = stats.startStage("boundaries");
LongObjectMap<PreparedGeometry> countryBoundaries = prepareRegionPolygons();
long number = 0;
@ -313,10 +315,16 @@ public class Boundary implements
@NotNull
private BorderingRegions getBorderingRegions(
LongObjectMap<PreparedGeometry> countryBoundaries,
Set<Long> regions,
Set<Long> allRegions,
LineString lineString
) {
Long rightCountry = null, leftCountry = null;
Set<Long> validRegions = allRegions.stream()
.filter(countryBoundaries::containsKey)
.collect(Collectors.toSet());
if (validRegions.isEmpty()) {
return BorderingRegions.empty();
}
List<Long> rights = new ArrayList<>();
List<Long> lefts = new ArrayList<>();
int steps = 10;
@ -324,7 +332,7 @@ public class Boundary implements
double ratio = (double) (i + 1) / (steps + 2);
Point right = GeoUtils.pointAlongOffset(lineString, ratio, COUNTRY_TEST_OFFSET);
Point left = GeoUtils.pointAlongOffset(lineString, ratio, -COUNTRY_TEST_OFFSET);
for (Long regionId : regions) {
for (Long regionId : validRegions) {
PreparedGeometry geom = countryBoundaries.get(regionId);
if (geom != null) {
if (geom.contains(right)) {
@ -347,7 +355,13 @@ public class Boundary implements
}
if (leftCountry == null && rightCountry == null) {
LOGGER.warn("[boundaries] no left or right country for border between country relations: " + regions);
Coordinate point = GeoUtils.worldToLatLonCoords(GeoUtils.pointAlongOffset(lineString, 0.5, 0)).getCoordinate();
LOGGER.warn("no left or right country for border between OSM country relations: %s around %.5f, %.5f"
.formatted(
validRegions,
point.getX(),
point.getY()
));
}
return new BorderingRegions(leftCountry, rightCountry);
@ -355,7 +369,7 @@ public class Boundary implements
@NotNull
private LongObjectMap<PreparedGeometry> prepareRegionPolygons() {
LOGGER.info("[boundaries] Creating polygons for " + regionGeometries.size() + " boundaries");
LOGGER.info("Creating polygons for " + regionGeometries.size() + " boundaries");
LongObjectMap<PreparedGeometry> countryBoundaries = new GHLongObjectHashMap<>();
for (var entry : regionGeometries.entrySet()) {
Long regionId = entry.getKey();
@ -364,15 +378,17 @@ public class Boundary implements
try {
Geometry combined = polygonizer.getGeometry().union();
if (combined.isEmpty()) {
LOGGER.warn("[boundaries] No valid polygons found for " + regionId);
LOGGER.warn("Unable to form closed polygon for OSM relation " + regionId
+ " (likely missing edges)");
} else {
countryBoundaries.put(regionId, PreparedGeometryFactory.prepare(combined));
}
} catch (TopologyException e) {
LOGGER.warn("[boundaries] Unable to build boundary polygon for " + regionId + ": " + e.getMessage());
LOGGER
.warn("Unable to build boundary polygon for OSM relation " + regionId + ": " + e.getMessage());
}
}
LOGGER.info("[boundaries] Finished creating " + countryBoundaries.size() + " country polygons");
LOGGER.info("Finished creating " + countryBoundaries.size() + " country polygons");
return countryBoundaries;
}
@ -383,7 +399,12 @@ public class Boundary implements
.orElse(null);
}
private static record BorderingRegions(Long left, Long right) {}
private static record BorderingRegions(Long left, Long right) {
public static BorderingRegions empty() {
return new BorderingRegions(null, null);
}
}
private static record BoundaryRelation(
long id,

Wyświetl plik

@ -193,6 +193,29 @@ public class BoundaryTest extends AbstractLayerTest {
Map.of())));
}
@Test
public void testOsmBoundaryLevelTwoAndAHalf() {
var relation = new ReaderRelation(1);
relation.setTag("type", "boundary");
relation.setTag("admin_level", "2.5");
relation.setTag("boundary", "administrative");
assertFeatures(14, List.of(Map.of(
"_layer", "boundary",
"_type", "line",
"disputed", 0,
"maritime", 0,
"admin_level", 3,
"_minzoom", 5,
"_maxzoom", 14,
"_buffer", 4d,
"_minpixelsize", 0d
)), process(lineFeatureWithRelation(
profile.preprocessOsmRelation(relation),
Map.of())));
}
@Test
public void testOsmBoundaryTakesMinAdminLevel() {
var relation1 = new ReaderRelation(1);

Wyświetl plik

@ -1,8 +0,0 @@
appenders=console
appender.console.type=Console
appender.console.name=STDOUT
appender.console.layout.type=PatternLayout
appender.console.layout.pattern=%d{HH:mm:ss.SSS} [%t] %.-3p - %m%n
rootLogger.level=warn
rootLogger.appenderRefs=stdout
rootLogger.appenderRef.stdout.ref=STDOUT