walterddr commented on code in PR #9962: URL: https://github.com/apache/pinot/pull/9962#discussion_r1046607720
########## pinot-query-runtime/src/main/java/org/apache/pinot/query/runtime/executor/RoundRobinScheduler.java: ########## @@ -58,12 +64,29 @@ public class RoundRobinScheduler implements OpChainScheduler { @VisibleForTesting final Set<MailboxIdentifier> _seenMail = new HashSet<>(); + public RoundRobinScheduler() { + this(DEFAULT_RELEASE_TIMEOUT); + } + + public RoundRobinScheduler(long releaseTimeout) { + this(releaseTimeout, System::currentTimeMillis); + } + + public RoundRobinScheduler(long releaseTimeoutMs, Supplier<Long> ticker) { + _releaseTimeout = releaseTimeoutMs; + _ticker = ticker; + } Review Comment: this is a good idea since we might not always want to timeout based on sys time. To further strengthen can we do `_ticker = () -> System.currentTimeMillis() + _releaseTimeoutMs` ? (e.g. putting _releaseTimeout as part of the _ticker? one thinking: we might need to do "remaining time scheduling" in the future (this is what the V1 engine does) - e.g. broker computed the remaining time based on how many time it already spent on planning and dispatch. and the remaining time is used for timeouts, with this change I can simply replace `operatorChain::getRemainingScheduledTime` once we implemented it. ########## pinot-query-runtime/src/main/java/org/apache/pinot/query/runtime/executor/OpChainSchedulerService.java: ########## @@ -98,9 +108,12 @@ public void runJob() { // not complete, needs to re-register for scheduling register(operatorChain, false); } else { - LOGGER.debug("({}): Completed {}", - operatorChain, - operatorChain.getStats()); + if (result.isErrorBlock()) { + LOGGER.error("({}): Completed erroneously {} {}", operatorChain, operatorChain.getStats(), + result.getDataBlock().getExceptions()); + } else { + LOGGER.debug("({}): Completed {}", operatorChain, operatorChain.getStats()); Review Comment: irrelevant to this PR. but complete ing of an entire opchain. can we do a INFO log? (this is once per stage yes?) ########## pinot-query-runtime/src/main/java/org/apache/pinot/query/service/QueryConfig.java: ########## @@ -59,6 +62,12 @@ public class QueryConfig { public static final String KEY_OF_SERVER_RESPONSE_STATUS_ERROR = "ERROR"; public static final String KEY_OF_SERVER_RESPONSE_STATUS_OK = "OK"; + /** + * Configuration keys for managing the scheduler + */ + public static final String KEY_OF_SCHEDULER_RELEASE_TIMEOUT_MS = "pinot.query.scheduler.release.timeout.ms"; + public static final long DEFAULT_SCHEDULER_RELEASE_TIMEOUT_MS = TimeUnit.MINUTES.toSeconds(1); Review Comment: could you comment on why this default value is being chosen. I am sensing it is related to the default timeout, but if someone set timeout = 120second. isn't this going to cause problem? -- This is an automated message from the Apache Git Service. To respond to the message, please log on to GitHub and use the URL above to go to the specific comment. To unsubscribe, e-mail: commits-unsubscr...@pinot.apache.org For queries about this service, please contact Infrastructure at: us...@infra.apache.org --------------------------------------------------------------------- To unsubscribe, e-mail: commits-unsubscr...@pinot.apache.org For additional commands, e-mail: commits-h...@pinot.apache.org