From notifications-return-608-archive-asf-public=cust-asf.ponee.io@zookeeper.apache.org Tue Jul 2 17:28:38 2019 Return-Path: X-Original-To: archive-asf-public@cust-asf.ponee.io Delivered-To: archive-asf-public@cust-asf.ponee.io Received: from mail.apache.org (hermes.apache.org [207.244.88.153]) by mx-eu-01.ponee.io (Postfix) with SMTP id EC608180763 for ; Tue, 2 Jul 2019 19:28:37 +0200 (CEST) Received: (qmail 98228 invoked by uid 500); 2 Jul 2019 17:28:37 -0000 Mailing-List: contact notifications-help@zookeeper.apache.org; run by ezmlm Precedence: bulk List-Help: List-Unsubscribe: List-Post: List-Id: Reply-To: dev@zookeeper.apache.org Delivered-To: mailing list notifications@zookeeper.apache.org Received: (qmail 98219 invoked by uid 99); 2 Jul 2019 17:28:37 -0000 Received: from ec2-52-202-80-70.compute-1.amazonaws.com (HELO gitbox.apache.org) (52.202.80.70) by apache.org (qpsmtpd/0.29) with ESMTP; Tue, 02 Jul 2019 17:28:37 +0000 From: GitBox To: notifications@zookeeper.apache.org Subject: [GitHub] [zookeeper] hanm commented on a change in pull request #986: ZOOKEEPER-3243: Add server-side request throttling Message-ID: <156208851393.24200.6205054163342643146.gitbox@gitbox.apache.org> Date: Tue, 02 Jul 2019 17:28:33 -0000 Content-Type: text/plain; charset=utf-8 Content-Transfer-Encoding: 8bit hanm commented on a change in pull request #986: ZOOKEEPER-3243: Add server-side request throttling URL: https://github.com/apache/zookeeper/pull/986#discussion_r299588541 ########## File path: zookeeper-server/src/main/java/org/apache/zookeeper/server/RequestThrottler.java ########## @@ -0,0 +1,273 @@ +/** + * Licensed to the Apache Software Foundation (ASF) under one + * or more contributor license agreements. See the NOTICE file + * distributed with this work for additional information + * regarding copyright ownership. The ASF licenses this file + * to you under the Apache License, Version 2.0 (the + * "License"); you may not use this file except in compliance + * with the License. You may obtain a copy of the License at + * + * http://www.apache.org/licenses/LICENSE-2.0 + * + * Unless required by applicable law or agreed to in writing, software + * distributed under the License is distributed on an "AS IS" BASIS, + * WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied. + * See the License for the specific language governing permissions and + * limitations under the License. + */ + +package org.apache.zookeeper.server; + +import java.util.concurrent.LinkedBlockingQueue; + +import edu.umd.cs.findbugs.annotations.SuppressFBWarnings; +import org.slf4j.Logger; +import org.slf4j.LoggerFactory; + +import org.apache.zookeeper.common.Time; + +/** + * When enabled, the RequestThrottler limits the number of outstanding requests + * currently submitted to the request processor pipeline. The throttler augments + * the limit imposed by the globalOutstandingLimit that is enforced + * by the connection layer ({@link NIOServerCnxn}, {@link NettyServerCnxn}). + * + * The connection layer limit applies backpressure against the TCP connection by + * disabling selection on connections once the request limit is reached. However, + * the connection layer always allows a connection to send at least one request + * before disabling selection on that connection. Thus, in a scenario with 40000 + * client connections, the total number of requests inflight may be as high as + * 40000 even if the globalOustandingLimit was set lower. + * + * The RequestThrottler addresses this issue by adding additional queueing. When + * enabled, client connections no longer submit requests directly to the request + * processor pipeline but instead to the RequestThrottler. The RequestThrottler + * is then responsible for issuing requests to the request processors, and + * enforces a separate maxRequests limit. If the total number of + * outstanding requests is higher than maxRequests, the throttler + * will continually stall for stallTime milliseconds until + * underlimit. + * + * The RequestThrottler can also optionally drop stale requests rather than + * submit them to the processor pipeline. A stale request is a request sent + * by a connection that is already closed, and/or a request whose latency + * will end up being higher than its associated session timeout. The notion + * of staleness is configurable, @see Request for more details. + * + * To ensure ordering guarantees, if a request is ever dropped from a connection + * that connection is closed and flagged as invalid. All subsequent requests + * inflight from that connection are then dropped as well. + */ +public class RequestThrottler extends ZooKeeperCriticalThread { + private static final Logger LOG = LoggerFactory.getLogger(RequestThrottler.class); + + private final LinkedBlockingQueue submittedRequests = + new LinkedBlockingQueue(); + + private final ZooKeeperServer zks; + private volatile boolean stopping; + private volatile boolean killed; + + /** + * enabled can only be set via system properties and not at runtime to + * ensure requests are never sent sometimes to the RequestThrottler and + * other times directly to a request processor, thus potentially reodering + * requests. + * + * This setting is designed as a code kill switch. In normal operation, + * enabled should be set to true. When enabled, maxRequests can be adjusted + * to enable/disable throttling at runtime. + */ + private static final boolean enabled = + ZooKeeperServer.getBooleanProp("zookeeper.request_throttle", true); + + /** + * The total number of outstanding requests allowed before the throttler + * starts stalling. + * + * When maxRequests = 0, throttling is disabled. + */ + private static volatile int maxRequests = + Integer.getInteger("zookeeper.request_throttle_max_requests", 0); + + /** + * The time (in milliseconds) this is the maximum time for which throttler + * thread may wait to be notified that it may proceed processing a request. + */ + private static volatile int stallTime = + Integer.getInteger("zookeeper.request_throttle_stall_time", 100); + + /** + * When true, the throttler will drop stale requests rather than issue + * them to the request pipeline. A stale request is a request sent by + * a connection that is now closed, and/or a request that will have a + * request latency higher than the sessionTimeout. The staleness of + * a request is tunable property, @see Request for details. + */ + private static volatile boolean dropStaleRequests = + ZooKeeperServer.getBooleanProp("zookeeper.request_throttle_drop_stale", true); + + public RequestThrottler(ZooKeeperServer zks) { + super("RequestThrottler", zks.getZooKeeperServerListener()); + this.zks = zks; + this.stopping = false; + this.killed = false; + } + + public static int getMaxRequests() { + return maxRequests; + } + + public static void setMaxRequests(int requests) { + maxRequests = requests; + } + + public static int getStallTime() { + return stallTime; + } + + public static void setStallTime(int time) { + stallTime = time; + } + + public static boolean getDropStaleRequests() { + return dropStaleRequests; + } + + public static void setDropStaleRequests(boolean drop) { + dropStaleRequests = drop; + } + + public static boolean enabled() { + return enabled; + } + + @Override + public void run() { + try { + while (true) { + Request request = submittedRequests.take(); + if (Request.requestOfDeath == request) { + break; + } + + if (request.mustDrop()) { + continue; + } + + // Throttling is disabled when maxRequests = 0 + if (maxRequests > 0) { + while (!killed) { + if (dropStaleRequests && request.isStale()) { + // Note: this will close the connection + dropRequest(request); + ServerMetrics.getMetrics().STALE_REQUESTS_DROPPED.add(1); + request = null; + break; + } + if (zks.getInProcess() < maxRequests) { + break; + } + throttleSleep(stallTime); + } + } + + if (killed) { Review comment: Assume while we are shutting down (`kill` is set to true), we have a large queue of requests, where each request is stale and must be dropped. With current code, the outer most `while (true) {` loop will try loop through every request in the queue, because L145 `if (request.mustDrop()) {` is evaluated as true for every request (which is stale). In this case, the check for `killed` in the inner while loop will not get executed. This was the reason I was suggesting to move the `if (killed) {` check before the `if (request.mustDrop()) {`. Does this example make sense? Albeit this example somewhat hypothetical, having `if (killed) {` check earlier sounds no harm either. ---------------------------------------------------------------- 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. For queries about this service, please contact Infrastructure at: users@infra.apache.org With regards, Apache Git Services