@@ -218,7 +218,7 @@ static class AuthData {
218
218
* If any request's response in not received in configured requestTimeout
219
219
* then it is assumed that the response packet is lost.
220
220
*/
221
- private long requestTimeout ;
221
+ private final long requestTimeout ;
222
222
223
223
ZKWatchManager getWatcherManager () {
224
224
return watchManager ;
@@ -286,6 +286,8 @@ static class Packet {
286
286
287
287
WatchDeregistration watchDeregistration ;
288
288
289
+ long deadline = Long .MAX_VALUE ;
290
+
289
291
/** Convenience ctor */
290
292
Packet (
291
293
RequestHeader requestHeader ,
@@ -414,7 +416,12 @@ public ClientCnxn(
414
416
415
417
this .sendThread = new SendThread (clientCnxnSocket );
416
418
this .eventThread = new EventThread ();
417
- initRequestTimeout ();
419
+ this .requestTimeout = clientConfig .getRequestTimeout ();
420
+ LOG .info (
421
+ "{} value is {}. feature enabled={}" ,
422
+ ZKClientConfig .ZOOKEEPER_REQUEST_TIMEOUT ,
423
+ requestTimeout ,
424
+ requestTimeout > 0 );
418
425
}
419
426
420
427
public void start () {
@@ -728,8 +735,6 @@ protected void finishPacket(Packet p) {
728
735
p .replyHeader .setErr (Code .OK .intValue ());
729
736
}
730
737
}
731
- } catch (KeeperException .NoWatcherException nwe ) {
732
- p .replyHeader .setErr (nwe .code ().intValue ());
733
738
} catch (KeeperException ke ) {
734
739
p .replyHeader .setErr (ke .code ().intValue ());
735
740
}
@@ -765,21 +770,26 @@ protected void onConnecting(InetSocketAddress addr) {
765
770
766
771
}
767
772
768
- private void conLossPacket (Packet p ) {
773
+ private Code abortPacket (Packet p , Code cause ) {
769
774
if (p .replyHeader == null ) {
770
- return ;
775
+ return cause ;
771
776
}
772
777
switch (state ) {
773
- case AUTH_FAILED :
774
- p .replyHeader .setErr (KeeperException .Code .AUTHFAILED .intValue ());
775
- break ;
776
- case CLOSED :
777
- p .replyHeader .setErr (KeeperException .Code .SESSIONEXPIRED .intValue ());
778
- break ;
779
- default :
780
- p .replyHeader .setErr (KeeperException . Code . CONNECTIONLOSS .intValue ());
778
+ case AUTH_FAILED :
779
+ p .replyHeader .setErr (KeeperException .Code .AUTHFAILED .intValue ());
780
+ break ;
781
+ case CLOSED :
782
+ p .replyHeader .setErr (KeeperException .Code .SESSIONEXPIRED .intValue ());
783
+ break ;
784
+ default :
785
+ p .replyHeader .setErr (cause .intValue ());
781
786
}
782
787
finishPacket (p );
788
+ return Code .CONNECTIONLOSS ;
789
+ }
790
+
791
+ private void conLossPacket (Packet p ) {
792
+ abortPacket (p , Code .CONNECTIONLOSS );
783
793
}
784
794
785
795
private volatile long lastZxid ;
@@ -852,6 +862,8 @@ void readResponse(ByteBuffer incomingBuffer) throws IOException {
852
862
853
863
replyHdr .deserialize (bbia , "header" );
854
864
switch (replyHdr .getXid ()) {
865
+ case SET_WATCHES_XID :
866
+ return ;
855
867
case PING_XID :
856
868
LOG .debug ("Got ping response for session id: 0x{} after {}ms." ,
857
869
Long .toHexString (sessionId ),
@@ -1116,6 +1128,39 @@ private void logStartConnect(InetSocketAddress addr) {
1116
1128
}
1117
1129
}
1118
1130
1131
+ private long requestDeadline () {
1132
+ if (requestTimeout == 0 ) {
1133
+ return Long .MAX_VALUE ;
1134
+ }
1135
+
1136
+ // The correctness of following code depends on several implementation details:
1137
+ // 1. Polling of outgoingQueue happens only in SendThread.
1138
+ // 2. Adding to pendingQueue happens only in SendThread.
1139
+ //
1140
+ // It is possible for netty socket to readResponse for first pendingQueue entry
1141
+ // while we are checking deadline for the same entry. So, it is possible that
1142
+ // a request was responded near deadline, but we disconnect the session. Given
1143
+ // that we are dealing with timeout, this should not be much matter.
1144
+ //
1145
+ // In long term, we should sequence all pendingQueue operations to SendThread.
1146
+
1147
+ Packet p ;
1148
+ synchronized (pendingQueue ) {
1149
+ p = pendingQueue .peek ();
1150
+ }
1151
+ if (p != null ) {
1152
+ return p .deadline ;
1153
+ }
1154
+
1155
+ for (Packet packet : outgoingQueue ) {
1156
+ if (packet .requestHeader != null && packet .requestHeader .getXid () >= 0 ) {
1157
+ return packet .deadline ;
1158
+ }
1159
+ }
1160
+
1161
+ return Long .MAX_VALUE ;
1162
+ }
1163
+
1119
1164
@ Override
1120
1165
@ SuppressFBWarnings ("JLM_JSR166_UTILCONCURRENT_MONITORENTER" )
1121
1166
public void run () {
@@ -1192,6 +1237,14 @@ public void run() {
1192
1237
LOG .warn (warnInfo );
1193
1238
throw new SessionTimeoutException (warnInfo );
1194
1239
}
1240
+ long deadline = requestDeadline ();
1241
+ if (deadline != Long .MAX_VALUE ) {
1242
+ long now = Time .currentElapsedTime ();
1243
+ if (now >= deadline ) {
1244
+ throw new KeeperException .RequestTimeoutException ();
1245
+ }
1246
+ to = Integer .min (to , (int ) (deadline - now ));
1247
+ }
1195
1248
if (state .isConnected ()) {
1196
1249
//1000(1 second) is to prevent race condition missing to send the second ping
1197
1250
//also make sure not to send too many pings when readTimeout is small
@@ -1240,9 +1293,14 @@ public void run() {
1240
1293
serverAddress ,
1241
1294
e );
1242
1295
1296
+ Code cause = Code .CONNECTIONLOSS ;
1297
+ if (e instanceof KeeperException ) {
1298
+ cause = ((KeeperException ) e ).code ();
1299
+ }
1300
+
1243
1301
// At this point, there might still be new packets appended to outgoingQueue.
1244
1302
// they will be handled in next connection or cleared up if closed.
1245
- cleanAndNotifyState ();
1303
+ cleanAndNotifyState (cause );
1246
1304
}
1247
1305
}
1248
1306
}
@@ -1268,8 +1326,8 @@ public void run() {
1268
1326
"SendThread exited loop for session: 0x" + Long .toHexString (getSessionId ()));
1269
1327
}
1270
1328
1271
- private void cleanAndNotifyState () {
1272
- cleanup ();
1329
+ private void cleanAndNotifyState (Code cause ) {
1330
+ cleanup (cause );
1273
1331
if (state .isAlive ()) {
1274
1332
eventThread .queueEvent (new WatchedEvent (Event .EventType .None , Event .KeeperState .Disconnected , null ));
1275
1333
}
@@ -1328,10 +1386,14 @@ private void pingRwServer() throws RWServerFoundException {
1328
1386
}
1329
1387
1330
1388
private void cleanup () {
1389
+ cleanup (Code .CONNECTIONLOSS );
1390
+ }
1391
+
1392
+ private void cleanup (Code cause ) {
1331
1393
clientCnxnSocket .cleanup ();
1332
1394
synchronized (pendingQueue ) {
1333
1395
for (Packet p : pendingQueue ) {
1334
- conLossPacket ( p );
1396
+ cause = abortPacket ( p , cause );
1335
1397
}
1336
1398
pendingQueue .clear ();
1337
1399
}
@@ -1341,7 +1403,7 @@ private void cleanup() {
1341
1403
Iterator <Packet > iter = outgoingQueue .iterator ();
1342
1404
while (iter .hasNext ()) {
1343
1405
Packet p = iter .next ();
1344
- conLossPacket ( p );
1406
+ cause = abortPacket ( p , cause );
1345
1407
iter .remove ();
1346
1408
}
1347
1409
}
@@ -1525,37 +1587,13 @@ public ReplyHeader submitRequest(
1525
1587
watchRegistration ,
1526
1588
watchDeregistration );
1527
1589
synchronized (packet ) {
1528
- if (requestTimeout > 0 ) {
1529
- // Wait for request completion with timeout
1530
- waitForPacketFinish (r , packet );
1531
- } else {
1532
- // Wait for request completion infinitely
1533
- while (!packet .finished ) {
1534
- packet .wait ();
1535
- }
1590
+ while (!packet .finished ) {
1591
+ packet .wait ();
1536
1592
}
1537
1593
}
1538
- if (r .getErr () == Code .REQUESTTIMEOUT .intValue ()) {
1539
- sendThread .cleanAndNotifyState ();
1540
- }
1541
1594
return r ;
1542
1595
}
1543
1596
1544
- /**
1545
- * Wait for request completion with timeout.
1546
- */
1547
- private void waitForPacketFinish (ReplyHeader r , Packet packet ) throws InterruptedException {
1548
- long waitStartTime = Time .currentElapsedTime ();
1549
- while (!packet .finished ) {
1550
- packet .wait (requestTimeout );
1551
- if (!packet .finished && ((Time .currentElapsedTime () - waitStartTime ) >= requestTimeout )) {
1552
- LOG .error ("Timeout error occurred for the packet '{}'." , packet );
1553
- r .setErr (Code .REQUESTTIMEOUT .intValue ());
1554
- break ;
1555
- }
1556
- }
1557
- }
1558
-
1559
1597
public void saslCompleted () {
1560
1598
sendThread .getClientCnxnSocket ().saslCompleted ();
1561
1599
}
@@ -1612,6 +1650,9 @@ public Packet queuePacket(
1612
1650
packet .clientPath = clientPath ;
1613
1651
packet .serverPath = serverPath ;
1614
1652
packet .watchDeregistration = watchDeregistration ;
1653
+ if (requestTimeout != 0 && h .getXid () >= 0 ) {
1654
+ packet .deadline = Time .currentElapsedTime () + requestTimeout ;
1655
+ }
1615
1656
// The synchronized block here is for two purpose:
1616
1657
// 1. synchronize with the final cleanup() in SendThread.run() to avoid race
1617
1658
// 2. synchronized against each packet. So if a closeSession packet is added,
@@ -1669,25 +1710,6 @@ public LocalCallback(AsyncCallback cb, int rc, String path, Object ctx) {
1669
1710
1670
1711
}
1671
1712
1672
- private void initRequestTimeout () {
1673
- try {
1674
- requestTimeout = clientConfig .getLong (
1675
- ZKClientConfig .ZOOKEEPER_REQUEST_TIMEOUT ,
1676
- ZKClientConfig .ZOOKEEPER_REQUEST_TIMEOUT_DEFAULT );
1677
- LOG .info (
1678
- "{} value is {}. feature enabled={}" ,
1679
- ZKClientConfig .ZOOKEEPER_REQUEST_TIMEOUT ,
1680
- requestTimeout ,
1681
- requestTimeout > 0 );
1682
- } catch (NumberFormatException e ) {
1683
- LOG .error (
1684
- "Configured value {} for property {} can not be parsed to long." ,
1685
- clientConfig .getProperty (ZKClientConfig .ZOOKEEPER_REQUEST_TIMEOUT ),
1686
- ZKClientConfig .ZOOKEEPER_REQUEST_TIMEOUT );
1687
- throw e ;
1688
- }
1689
- }
1690
-
1691
1713
public ZooKeeperSaslClient getZooKeeperSaslClient () {
1692
1714
return sendThread .getZooKeeperSaslClient ();
1693
1715
}
0 commit comments