[ https://issues.apache.org/jira/browse/GEODE-8004?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=17097335#comment-17097335 ]
ASF GitHub Bot commented on GEODE-8004: --------------------------------------- jujoramos commented on a change in pull request #4978: URL: https://github.com/apache/geode/pull/4978#discussion_r418501644 ########## File path: geode-core/src/distributedTest/java/org/apache/geode/internal/cache/tier/sockets/PingOpDistributedTest.java ########## @@ -0,0 +1,202 @@ +/* + * 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.geode.internal.cache.tier.sockets; + +import static java.util.Arrays.asList; +import static org.apache.geode.internal.AvailablePortHelper.getRandomAvailableTCPPortsForDUnitSite; +import static org.apache.geode.test.dunit.VM.getVM; +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; + +import java.io.IOException; +import java.io.Serializable; +import java.util.Collections; +import java.util.List; + +import org.junit.Before; +import org.junit.Rule; +import org.junit.Test; + +import org.apache.geode.cache.client.ClientCacheFactory; +import org.apache.geode.cache.client.PoolFactory; +import org.apache.geode.cache.client.PoolManager; +import org.apache.geode.cache.client.ServerOperationException; +import org.apache.geode.cache.client.internal.PingOp; +import org.apache.geode.cache.client.internal.PoolImpl; +import org.apache.geode.cache.server.CacheServer; +import org.apache.geode.distributed.internal.ServerLocation; +import org.apache.geode.distributed.internal.membership.InternalDistributedMember; +import org.apache.geode.test.awaitility.GeodeAwaitility; +import org.apache.geode.test.dunit.VM; +import org.apache.geode.test.dunit.rules.CacheRule; +import org.apache.geode.test.dunit.rules.ClientCacheRule; +import org.apache.geode.test.dunit.rules.DistributedRule; +import org.apache.geode.test.junit.rules.serializable.SerializableTemporaryFolder; +import org.apache.geode.test.junit.rules.serializable.SerializableTestName; + +public class PingOpDistributedTest implements Serializable { + + @Rule + public CacheRule cacheRule = new CacheRule(); + + @Rule + public ClientCacheRule clientCacheRule = new ClientCacheRule(); + + @Rule + public SerializableTestName testName = new SerializableTestName(); + + @Rule + public DistributedRule distributedRule = new DistributedRule(2); + + @Rule + public SerializableTemporaryFolder folder = new SerializableTemporaryFolder(); + + private VM client; + private VM server1, server2; + private int server1Port, server2Port; + + + private void initServer(int serverPort) throws IOException { + cacheRule.createCache(); + CacheServer cacheServer = cacheRule.getCache().addCacheServer(); + cacheServer.setPort(serverPort); + + // "Disable" the auto-ping for the duration of this test. + cacheServer.setMaximumTimeBetweenPings((int) GeodeAwaitility.getTimeout().toMillis()); + cacheServer.start(); + } + + private void initClient(String poolName, List<Integer> serverPorts) { + final ClientCacheFactory clientCacheFactory = new ClientCacheFactory(); + clientCacheFactory.create(); + + PoolFactory poolFactory = PoolManager.createFactory(); + serverPorts.forEach(serverPort -> poolFactory.addServer("localhost", serverPort)); + + // "Disable" the auto-ping for the duration of this test. + poolFactory.setPingInterval((int) GeodeAwaitility.getTimeout().toMillis()); + poolFactory.create(poolName); + } + + @Before + public void setUp() throws IOException { + int[] ports = getRandomAvailableTCPPortsForDUnitSite(2); + + client = getVM(0); + server1 = getVM(1); + server2 = getVM(2); + server1Port = ports[0]; + server2Port = ports[1]; + server1.invoke(() -> initServer(server1Port)); + server2.invoke(() -> initServer(server2Port)); + } + + void parametrizedSetUp(String poolName, List<Integer> serverPorts) { + client.invoke(() -> initClient(poolName, serverPorts)); + } + + public void executePing(String poolName, int serverPort, + InternalDistributedMember distributedMember) { + PoolImpl poolImpl = (PoolImpl) PoolManager.find(poolName); + PingOp.execute(poolImpl, new ServerLocation("localhost", serverPort), distributedMember); + } + + public Long getSingleHeartBeat() { + ClientHealthMonitor chm = ClientHealthMonitor.getInstance(); + if (chm.getClientHeartbeats().size() == 0) { + return 0L; + } + assertThat(chm.getClientHeartbeats()).isNotEmpty().hasSize(1); + + return chm.getClientHeartbeats().entrySet().iterator().next().getValue(); + } + + @Test + public void regularPingFlow() { + final String poolName = testName.getMethodName(); + parametrizedSetUp(poolName, Collections.singletonList(server1Port)); + InternalDistributedMember distributedMember1 = (InternalDistributedMember) server1 + .invoke(() -> cacheRule.getCache().getDistributedSystem().getDistributedMember()); + + client.invoke(() -> executePing(poolName, server1Port, distributedMember1)); + Long firstHeartbeat = server1.invoke(this::getSingleHeartBeat); + + client.invoke(() -> executePing(poolName, server1Port, distributedMember1)); + Long secondHeartbeat = server1.invoke(this::getSingleHeartBeat); + + assertThat(secondHeartbeat).isGreaterThan(firstHeartbeat); + + } + + @Test + public void memberShouldNotRedirectPingMessageWhenClientCachedViewIdIsWrong() throws IOException { Review comment: The exception is not thrown anywhere in the method, it can be removed. ########## File path: geode-core/src/distributedTest/java/org/apache/geode/internal/cache/tier/sockets/PingOpDistributedTest.java ########## @@ -0,0 +1,202 @@ +/* + * 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.geode.internal.cache.tier.sockets; + +import static java.util.Arrays.asList; +import static org.apache.geode.internal.AvailablePortHelper.getRandomAvailableTCPPortsForDUnitSite; +import static org.apache.geode.test.dunit.VM.getVM; +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; + +import java.io.IOException; +import java.io.Serializable; +import java.util.Collections; +import java.util.List; + +import org.junit.Before; +import org.junit.Rule; +import org.junit.Test; + +import org.apache.geode.cache.client.ClientCacheFactory; +import org.apache.geode.cache.client.PoolFactory; +import org.apache.geode.cache.client.PoolManager; +import org.apache.geode.cache.client.ServerOperationException; +import org.apache.geode.cache.client.internal.PingOp; +import org.apache.geode.cache.client.internal.PoolImpl; +import org.apache.geode.cache.server.CacheServer; +import org.apache.geode.distributed.internal.ServerLocation; +import org.apache.geode.distributed.internal.membership.InternalDistributedMember; +import org.apache.geode.test.awaitility.GeodeAwaitility; +import org.apache.geode.test.dunit.VM; +import org.apache.geode.test.dunit.rules.CacheRule; +import org.apache.geode.test.dunit.rules.ClientCacheRule; +import org.apache.geode.test.dunit.rules.DistributedRule; +import org.apache.geode.test.junit.rules.serializable.SerializableTemporaryFolder; +import org.apache.geode.test.junit.rules.serializable.SerializableTestName; + +public class PingOpDistributedTest implements Serializable { + + @Rule + public CacheRule cacheRule = new CacheRule(); + + @Rule + public ClientCacheRule clientCacheRule = new ClientCacheRule(); + + @Rule + public SerializableTestName testName = new SerializableTestName(); + + @Rule + public DistributedRule distributedRule = new DistributedRule(2); + + @Rule + public SerializableTemporaryFolder folder = new SerializableTemporaryFolder(); + + private VM client; + private VM server1, server2; + private int server1Port, server2Port; + + + private void initServer(int serverPort) throws IOException { + cacheRule.createCache(); + CacheServer cacheServer = cacheRule.getCache().addCacheServer(); + cacheServer.setPort(serverPort); + + // "Disable" the auto-ping for the duration of this test. + cacheServer.setMaximumTimeBetweenPings((int) GeodeAwaitility.getTimeout().toMillis()); + cacheServer.start(); + } + + private void initClient(String poolName, List<Integer> serverPorts) { + final ClientCacheFactory clientCacheFactory = new ClientCacheFactory(); + clientCacheFactory.create(); + + PoolFactory poolFactory = PoolManager.createFactory(); + serverPorts.forEach(serverPort -> poolFactory.addServer("localhost", serverPort)); + + // "Disable" the auto-ping for the duration of this test. + poolFactory.setPingInterval((int) GeodeAwaitility.getTimeout().toMillis()); + poolFactory.create(poolName); + } + + @Before + public void setUp() throws IOException { + int[] ports = getRandomAvailableTCPPortsForDUnitSite(2); + + client = getVM(0); + server1 = getVM(1); + server2 = getVM(2); + server1Port = ports[0]; + server2Port = ports[1]; + server1.invoke(() -> initServer(server1Port)); + server2.invoke(() -> initServer(server2Port)); + } + + void parametrizedSetUp(String poolName, List<Integer> serverPorts) { + client.invoke(() -> initClient(poolName, serverPorts)); + } + + public void executePing(String poolName, int serverPort, + InternalDistributedMember distributedMember) { + PoolImpl poolImpl = (PoolImpl) PoolManager.find(poolName); + PingOp.execute(poolImpl, new ServerLocation("localhost", serverPort), distributedMember); + } + + public Long getSingleHeartBeat() { + ClientHealthMonitor chm = ClientHealthMonitor.getInstance(); + if (chm.getClientHeartbeats().size() == 0) { + return 0L; + } + assertThat(chm.getClientHeartbeats()).isNotEmpty().hasSize(1); + + return chm.getClientHeartbeats().entrySet().iterator().next().getValue(); + } + + @Test + public void regularPingFlow() { + final String poolName = testName.getMethodName(); + parametrizedSetUp(poolName, Collections.singletonList(server1Port)); + InternalDistributedMember distributedMember1 = (InternalDistributedMember) server1 + .invoke(() -> cacheRule.getCache().getDistributedSystem().getDistributedMember()); + + client.invoke(() -> executePing(poolName, server1Port, distributedMember1)); + Long firstHeartbeat = server1.invoke(this::getSingleHeartBeat); + + client.invoke(() -> executePing(poolName, server1Port, distributedMember1)); + Long secondHeartbeat = server1.invoke(this::getSingleHeartBeat); + + assertThat(secondHeartbeat).isGreaterThan(firstHeartbeat); + + } + + @Test + public void memberShouldNotRedirectPingMessageWhenClientCachedViewIdIsWrong() throws IOException { + final String poolName = testName.getMethodName(); + parametrizedSetUp(poolName, Collections.singletonList(server1Port)); + InternalDistributedMember distributedMember1 = (InternalDistributedMember) server1 + .invoke(() -> cacheRule.getCache().getDistributedSystem().getDistributedMember()); + + client.invoke(() -> { + PoolImpl poolImpl = (PoolImpl) PoolManager.find(poolName); + distributedMember1.setVmViewId(distributedMember1.getVmViewId() + 1); + assertThatThrownBy(() -> { + PingOp.execute(poolImpl, new ServerLocation("localhost", server1Port), distributedMember1); + }).isInstanceOf(ServerOperationException.class).hasMessageContaining("has different viewId:"); + }); + } + + @Test + public void pingReturnsErrorIfTheTargetServerIsNotAMember() throws IOException { Review comment: The exception is not thrown anywhere in the method, it can be removed. ########## File path: geode-core/src/distributedTest/java/org/apache/geode/internal/cache/tier/sockets/PingOpDistributedTest.java ########## @@ -0,0 +1,202 @@ +/* + * 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.geode.internal.cache.tier.sockets; + +import static java.util.Arrays.asList; +import static org.apache.geode.internal.AvailablePortHelper.getRandomAvailableTCPPortsForDUnitSite; +import static org.apache.geode.test.dunit.VM.getVM; +import static org.assertj.core.api.Assertions.assertThat; +import static org.assertj.core.api.Assertions.assertThatThrownBy; + +import java.io.IOException; +import java.io.Serializable; +import java.util.Collections; +import java.util.List; + +import org.junit.Before; +import org.junit.Rule; +import org.junit.Test; + +import org.apache.geode.cache.client.ClientCacheFactory; +import org.apache.geode.cache.client.PoolFactory; +import org.apache.geode.cache.client.PoolManager; +import org.apache.geode.cache.client.ServerOperationException; +import org.apache.geode.cache.client.internal.PingOp; +import org.apache.geode.cache.client.internal.PoolImpl; +import org.apache.geode.cache.server.CacheServer; +import org.apache.geode.distributed.internal.ServerLocation; +import org.apache.geode.distributed.internal.membership.InternalDistributedMember; +import org.apache.geode.test.awaitility.GeodeAwaitility; +import org.apache.geode.test.dunit.VM; +import org.apache.geode.test.dunit.rules.CacheRule; +import org.apache.geode.test.dunit.rules.ClientCacheRule; +import org.apache.geode.test.dunit.rules.DistributedRule; +import org.apache.geode.test.junit.rules.serializable.SerializableTemporaryFolder; +import org.apache.geode.test.junit.rules.serializable.SerializableTestName; + +public class PingOpDistributedTest implements Serializable { + + @Rule + public CacheRule cacheRule = new CacheRule(); + + @Rule + public ClientCacheRule clientCacheRule = new ClientCacheRule(); + + @Rule + public SerializableTestName testName = new SerializableTestName(); + + @Rule + public DistributedRule distributedRule = new DistributedRule(2); + + @Rule + public SerializableTemporaryFolder folder = new SerializableTemporaryFolder(); + + private VM client; + private VM server1, server2; + private int server1Port, server2Port; + + + private void initServer(int serverPort) throws IOException { + cacheRule.createCache(); + CacheServer cacheServer = cacheRule.getCache().addCacheServer(); + cacheServer.setPort(serverPort); + + // "Disable" the auto-ping for the duration of this test. + cacheServer.setMaximumTimeBetweenPings((int) GeodeAwaitility.getTimeout().toMillis()); + cacheServer.start(); + } + + private void initClient(String poolName, List<Integer> serverPorts) { + final ClientCacheFactory clientCacheFactory = new ClientCacheFactory(); + clientCacheFactory.create(); + + PoolFactory poolFactory = PoolManager.createFactory(); + serverPorts.forEach(serverPort -> poolFactory.addServer("localhost", serverPort)); + + // "Disable" the auto-ping for the duration of this test. + poolFactory.setPingInterval((int) GeodeAwaitility.getTimeout().toMillis()); + poolFactory.create(poolName); + } + + @Before + public void setUp() throws IOException { + int[] ports = getRandomAvailableTCPPortsForDUnitSite(2); + + client = getVM(0); + server1 = getVM(1); + server2 = getVM(2); + server1Port = ports[0]; + server2Port = ports[1]; + server1.invoke(() -> initServer(server1Port)); + server2.invoke(() -> initServer(server2Port)); + } + + void parametrizedSetUp(String poolName, List<Integer> serverPorts) { + client.invoke(() -> initClient(poolName, serverPorts)); + } + + public void executePing(String poolName, int serverPort, + InternalDistributedMember distributedMember) { + PoolImpl poolImpl = (PoolImpl) PoolManager.find(poolName); + PingOp.execute(poolImpl, new ServerLocation("localhost", serverPort), distributedMember); + } + + public Long getSingleHeartBeat() { + ClientHealthMonitor chm = ClientHealthMonitor.getInstance(); + if (chm.getClientHeartbeats().size() == 0) { + return 0L; + } + assertThat(chm.getClientHeartbeats()).isNotEmpty().hasSize(1); + + return chm.getClientHeartbeats().entrySet().iterator().next().getValue(); + } + + @Test + public void regularPingFlow() { + final String poolName = testName.getMethodName(); + parametrizedSetUp(poolName, Collections.singletonList(server1Port)); + InternalDistributedMember distributedMember1 = (InternalDistributedMember) server1 + .invoke(() -> cacheRule.getCache().getDistributedSystem().getDistributedMember()); + + client.invoke(() -> executePing(poolName, server1Port, distributedMember1)); + Long firstHeartbeat = server1.invoke(this::getSingleHeartBeat); + + client.invoke(() -> executePing(poolName, server1Port, distributedMember1)); + Long secondHeartbeat = server1.invoke(this::getSingleHeartBeat); + + assertThat(secondHeartbeat).isGreaterThan(firstHeartbeat); + + } + + @Test + public void memberShouldNotRedirectPingMessageWhenClientCachedViewIdIsWrong() throws IOException { + final String poolName = testName.getMethodName(); + parametrizedSetUp(poolName, Collections.singletonList(server1Port)); + InternalDistributedMember distributedMember1 = (InternalDistributedMember) server1 + .invoke(() -> cacheRule.getCache().getDistributedSystem().getDistributedMember()); + + client.invoke(() -> { + PoolImpl poolImpl = (PoolImpl) PoolManager.find(poolName); + distributedMember1.setVmViewId(distributedMember1.getVmViewId() + 1); + assertThatThrownBy(() -> { + PingOp.execute(poolImpl, new ServerLocation("localhost", server1Port), distributedMember1); + }).isInstanceOf(ServerOperationException.class).hasMessageContaining("has different viewId:"); + }); + } + + @Test + public void pingReturnsErrorIfTheTargetServerIsNotAMember() throws IOException { + final String poolName = testName.getMethodName(); + parametrizedSetUp(poolName, Collections.singletonList(server1Port)); + int notUsedPort = getRandomAvailableTCPPortsForDUnitSite(1)[0]; + InternalDistributedMember fakeDistributedMember = + new InternalDistributedMember("localhost", notUsedPort); + client.invoke(() -> { + PoolImpl poolImpl = (PoolImpl) PoolManager.find(poolName); + assertThatThrownBy(() -> { + PingOp.execute(poolImpl, new ServerLocation("localhost", server1Port), + fakeDistributedMember); + }).isInstanceOf(ServerOperationException.class) + .hasMessageContaining("Unable to ping non-member"); + }); + } + + @Test + public void memberShouldCorrectlyRedirectPingMessage() throws IOException, InterruptedException { Review comment: The exceptions are not thrown anywhere in the method, they can be removed. ########## File path: geode-core/src/main/java/org/apache/geode/internal/cache/tier/sockets/command/Ping.java ########## @@ -47,10 +50,37 @@ public void cmdExecute(final Message clientMessage, final ServerConnection serve clientMessage.getTransactionId(), serverConnection.getSocketString(), (DistributionStats.getStatTime() - start)); } + if (clientMessage.getNumberOfParts() > 0) { + try { + InternalDistributedMember targetServer = + (InternalDistributedMember) clientMessage.getPart(0).getObject(); + InternalDistributedMember myID = serverConnection.getCache().getMyId(); + if (!myID.equals(targetServer)) { + if (myID.compareTo(targetServer, true, false) == 0) { + String errorMessage = + String.format("Target server " + targetServer + " has different viewId: " + myID); Review comment: No need to use `format` as we are not using parameters, just concatenating `String` objects. ########## File path: geode-core/src/main/java/org/apache/geode/distributed/internal/ServerLocationAndMemberId.java ########## @@ -0,0 +1,72 @@ +/* + * 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.geode.distributed.internal; + +public class ServerLocationAndMemberId { + + private ServerLocation serverLocation; + private String memberId; Review comment: Can these two fields be declared as `final`?. ########## File path: geode-core/src/main/java/org/apache/geode/cache/client/internal/PingOp.java ########## @@ -65,17 +71,33 @@ protected boolean needsUserId() { @Override protected void sendMessage(Connection cnx) throws Exception { getMessage().clearMessageHasSecurePartFlag(); - this.startTime = System.currentTimeMillis(); - getMessage().send(false); + getMessage().setNumberOfParts(1); + getMessage().addObjPart(serverID); + getMessage().send(true); Message.MESSAGE_TYPE.set(MessageType.PING); } @Override protected Object processResponse(Message msg) throws Exception { - processAck(msg, "ping"); + processAck(msg); return null; } + private void processAck(Message msg) throws Exception { + final int msgType = msg.getMessageType(); + if (msgType != MessageType.REPLY) { + Part part = msg.getPart(0); + if (msgType == MessageType.EXCEPTION) { + Throwable t = (Throwable) part.getObject(); + throw new ServerOperationException("While performing a remote ping: " + t.getMessage(), + t); + } else { + throw new InternalGemFireError( + "Unexpected message type " + MessageType.getString(msgType)); + } + } + } Review comment: I'm not sure it is entirely correct to override the `procesAck` method... Looking through the source code, there's only one operation overriding this (`TXSynchronizationOp`) and it ends up delegating to the super class method whenever the exception is not the one it knows how to handle. There's also a comment regarding `c++` clients in `AbstractOp.processAck()`, we should be careful not to break those clients as well. @bschuchardt: thoughts on this one?. ########## File path: geode-core/src/main/java/org/apache/geode/internal/cache/GridAdvisor.java ########## @@ -418,18 +418,24 @@ public String toString() { public int hashCode() { final String thisHost = this.gp.getHost(); final int thisPort = this.gp.getPort(); - return thisHost != null ? (thisHost.hashCode() ^ thisPort) : thisPort; + final String thisMemberId = this.getMemberId().getUniqueId(); + final int thisMemberIdHashCode = (thisMemberId != null) ? thisMemberId.hashCode() : 0; + return thisHost != null ? (thisHost.hashCode() ^ thisPort) + thisMemberIdHashCode + : thisPort + thisMemberIdHashCode; } @Override public boolean equals(Object obj) { if (obj instanceof GridProfileId) { final GridProfileId other = (GridProfileId) obj; + if (this.gp.getPort() == other.gp.getPort()) { final String thisHost = this.gp.getHost(); final String otherHost = other.gp.getHost(); if (thisHost != null) { - return thisHost.equals(otherHost); + if (thisHost.equals(otherHost)) { + return this.getMemberId().equals(other.getMemberId()); Review comment: Should we use `getMemberId().getUniqueId()` here instead of just `getMemberId()`, as that's what we are using to calculate the `hashCode`?. ########## File path: geode-core/src/test/java/org/apache/geode/internal/cache/tier/sockets/command/PingTest.java ########## @@ -0,0 +1,136 @@ +/* + * 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.geode.internal.cache.tier.sockets.command; + +import static org.mockito.ArgumentMatchers.eq; +import static org.mockito.Mockito.mock; +import static org.mockito.Mockito.times; +import static org.mockito.Mockito.verify; +import static org.mockito.Mockito.when; + +import org.junit.Before; +import org.junit.Test; +import org.mockito.InjectMocks; +import org.mockito.Mock; +import org.mockito.MockitoAnnotations; + +import org.apache.geode.CancelCriterion; +import org.apache.geode.distributed.internal.DistributionManager; +import org.apache.geode.distributed.internal.ServerLocation; +import org.apache.geode.distributed.internal.membership.InternalDistributedMember; +import org.apache.geode.internal.cache.InternalCache; +import org.apache.geode.internal.cache.tier.sockets.Message; +import org.apache.geode.internal.cache.tier.sockets.Part; +import org.apache.geode.internal.cache.tier.sockets.ServerConnection; + +public class PingTest { Review comment: Tests could be improved to also assert that the receivedPing method from the ClientHealthMonitor class has been invoked when required. Just a suggestion, anyways. ---------------------------------------------------------------- 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: us...@infra.apache.org > Regression Introduced Through GEODE-7565 > ---------------------------------------- > > Key: GEODE-8004 > URL: https://issues.apache.org/jira/browse/GEODE-8004 > Project: Geode > Issue Type: Bug > Components: client/server > Reporter: Juan Ramos > Assignee: Juan Ramos > Priority: Major > Labels: GeodeCommons > > Intermittent errors were observed while executing some internal tests and > commit > [dd23ee8|https://github.com/apache/geode/commit/dd23ee8200cba67cea82e57e2e4ccedcdf9e8266] > was determined to be responsible. As of yet, no local reproduction of the > issue is available, but work is ongoing to provide a test that can be used to > debug the issue (a [PR|https://github.com/apache/geode/pull/4974] to revert > of the original commit has been opened and will be merged shortly, though, > this ticket is to investigate the root cause so the original commit can be > merged again into {{develop}}). > --- > It seems that a server is trying to read an {{ack}} response and, instead, it > receives a {{PING}} message: > {noformat} > [error 2020/04/18 23:44:22.758 PDT <poolTimer-edgeDescript-31> tid=0x165] > Unexpected error in pool task > <org.apache.geode.cache.client.internal.LiveServerPinger$PingTask@3483b110> > org.apache.geode.InternalGemFireError: Unexpected message type PING > at > org.apache.geode.cache.client.internal.AbstractOp.processAck(AbstractOp.java:264) > at > org.apache.geode.cache.client.internal.PingOp$PingOpImpl.processResponse(PingOp.java:82) > at > org.apache.geode.cache.client.internal.AbstractOp.processResponse(AbstractOp.java:222) > at > org.apache.geode.cache.client.internal.AbstractOp.attemptReadResponse(AbstractOp.java:207) > at > org.apache.geode.cache.client.internal.AbstractOp.attempt(AbstractOp.java:382) > at > org.apache.geode.cache.client.internal.ConnectionImpl.execute(ConnectionImpl.java:268) > at > org.apache.geode.cache.client.internal.pooling.PooledConnection.execute(PooledConnection.java:352) > at > org.apache.geode.cache.client.internal.OpExecutorImpl.executeWithPossibleReAuthentication(OpExecutorImpl.java:753) > at > org.apache.geode.cache.client.internal.OpExecutorImpl.executeOnServer(OpExecutorImpl.java:332) > at > org.apache.geode.cache.client.internal.OpExecutorImpl.executeOn(OpExecutorImpl.java:303) > at > org.apache.geode.cache.client.internal.PoolImpl.executeOn(PoolImpl.java:839) > at org.apache.geode.cache.client.internal.PingOp.execute(PingOp.java:38) > at > org.apache.geode.cache.client.internal.LiveServerPinger$PingTask.run2(LiveServerPinger.java:90) > at > org.apache.geode.cache.client.internal.PoolImpl$PoolTask.run(PoolImpl.java:1329) > at > java.util.concurrent.Executors$RunnableAdapter.call(Executors.java:511) > at java.util.concurrent.FutureTask.runAndReset(FutureTask.java:308) > at > org.apache.geode.internal.ScheduledThreadPoolExecutorWithKeepAlive$DelegatingScheduledFuture.run(ScheduledThreadPoolExecutorWithKeepAlive.java:276) > at > java.util.concurrent.ThreadPoolExecutor.runWorker(ThreadPoolExecutor.java:1149) > at > java.util.concurrent.ThreadPoolExecutor$Worker.run(ThreadPoolExecutor.java:624) > at java.lang.Thread.run(Thread.java:748) > {noformat} > Around the same time, another member of the distributed system logs the > following warning, which seems to be related to the original changes as well: > {noformat} > [warn 2020/04/18 23:44:22.757 PDT <ServerConnection on port 29019 Thread 1> > tid=0x298] Unable to ping non-member > rs-FullRegression19040559a2i32xlarge-hydra-client-63(bridgegemfire1_host1_4749:4749)<ec><v39>:41003 > for client > identity(rs-FullRegression19040559a2i32xlarge-hydra-client-63(edgegemfire3_host1_1071:1071:loner):50046:5a182991:edgegemfire3_host1_1071,connection=2 > {noformat} -- This message was sent by Atlassian Jira (v8.3.4#803005)