You are viewing a plain text version of this content. The canonical link for it is here.
Posted to jira@kafka.apache.org by GitBox <gi...@apache.org> on 2022/06/23 03:46:23 UTC

[GitHub] [kafka] C0urante commented on a diff in pull request #12320: KAFKA-13702: Connect RestClient overrides response status code on request failure

C0urante commented on code in PR #12320:
URL: https://github.com/apache/kafka/pull/12320#discussion_r904481428


##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);
+    }
+
+    @BeforeEach
+    public void mockSetup() {
+        httpClient = niceMock(HttpClient.class);
+    }
+
+    @Test
+    public void testSuccess() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.OK.getStatusCode();
+        String expectedResponse = toJsonString(new TestDTO("someContent"));
+        setupHttpClient(statusCode, expectedResponse);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertEquals(toJsonString(httpResp.body()), expectedResponse);
+    }
+
+    @Test
+    public void testNoContent() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.NO_CONTENT.getStatusCode();
+        setupHttpClient(statusCode, null);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertNull(httpResp.body());
+    }
+
+    @Test
+    public void testError() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.CONFLICT.getStatusCode();
+        ErrorMessage errorMsg = new ErrorMessage(Response.Status.GONE.getStatusCode(), "Some Error Message");
+        setupHttpClient(statusCode, toJsonString(errorMsg));
+        ConnectRestException e = assertThrows(ConnectRestException.class, () -> httpRequest(httpClient, TEST_TYPE));
+        assertEquals(e.statusCode(), statusCode);
+        assertEquals(e.errorCode(), errorMsg.errorCode());
+        assertEquals(e.getMessage(), errorMsg.message());
+    }
+
+    private void setupHttpClient(int responseCode, String responseJsonString) throws ExecutionException, InterruptedException, TimeoutException {
+        WorkerConfig workerConf = niceMock(WorkerConfig.class);
+        expect(workerConf.originals()).andReturn(Collections.emptyMap());

Review Comment:
   Why are we setting up this mock? As far as I can tell it's not used where.



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);
+    }
+
+    @BeforeEach
+    public void mockSetup() {
+        httpClient = niceMock(HttpClient.class);
+    }
+
+    @Test
+    public void testSuccess() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.OK.getStatusCode();
+        String expectedResponse = toJsonString(new TestDTO("someContent"));
+        setupHttpClient(statusCode, expectedResponse);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertEquals(toJsonString(httpResp.body()), expectedResponse);
+    }
+
+    @Test
+    public void testNoContent() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.NO_CONTENT.getStatusCode();
+        setupHttpClient(statusCode, null);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertNull(httpResp.body());
+    }
+
+    @Test
+    public void testError() throws ExecutionException, InterruptedException, TimeoutException {

Review Comment:
   This test case is about more than just handling errors in the client, right? We might want to rename it to something like `testErrorResponseStatusGetsPreserved` and then possibly add other cases that cover `IOException`s, unexpected (i.e, 1XX or 3XX) HTTP status codes, failures during request signing, and other possible `RuntimeException`s that may be encountered in the body of `RestClient::httpRequest`.



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);
+    }
+
+    @BeforeEach
+    public void mockSetup() {
+        httpClient = niceMock(HttpClient.class);
+    }
+
+    @Test
+    public void testSuccess() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.OK.getStatusCode();
+        String expectedResponse = toJsonString(new TestDTO("someContent"));
+        setupHttpClient(statusCode, expectedResponse);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertEquals(toJsonString(httpResp.body()), expectedResponse);
+    }
+
+    @Test
+    public void testNoContent() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.NO_CONTENT.getStatusCode();
+        setupHttpClient(statusCode, null);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertNull(httpResp.body());
+    }
+
+    @Test
+    public void testError() throws ExecutionException, InterruptedException, TimeoutException {

Review Comment:
   It'd also be fine to leave these other cases out for now since you've strictly increased the coverage for this class with the cases you've added already, but I do think it'd still be worth adding specificity to the name of the test case here just to illustrate to future maintainers what the test covers and what it does not.



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);
+    }
+
+    @BeforeEach
+    public void mockSetup() {
+        httpClient = niceMock(HttpClient.class);
+    }
+
+    @Test
+    public void testSuccess() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.OK.getStatusCode();
+        String expectedResponse = toJsonString(new TestDTO("someContent"));
+        setupHttpClient(statusCode, expectedResponse);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertEquals(toJsonString(httpResp.body()), expectedResponse);

Review Comment:
   It'd give us better coverage if we performed an assertion against the actual `TestDTO` object in the response, instead of the JSON-serialized string of it.



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);
+    }
+
+    @BeforeEach
+    public void mockSetup() {
+        httpClient = niceMock(HttpClient.class);
+    }
+
+    @Test
+    public void testSuccess() throws ExecutionException, InterruptedException, TimeoutException {

Review Comment:
   Nit: can just replace this all with `throws Exception`; makes it easier to modify these tests in the future if the set of expected exceptions changes.



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);

Review Comment:
   Nit: it's a little strange that so many of these parameters are `null`. Wouldn't it be more realistic to give real values here, even if they're just constants for the sake of testing?



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);
+    }
+
+    @BeforeEach
+    public void mockSetup() {
+        httpClient = niceMock(HttpClient.class);
+    }
+
+    @Test
+    public void testSuccess() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.OK.getStatusCode();
+        String expectedResponse = toJsonString(new TestDTO("someContent"));
+        setupHttpClient(statusCode, expectedResponse);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);

Review Comment:
   Nit: can we flip the order of arguments here and elsewhere? The `assertEquals` method should be called with the expected value as the first argument, and the actual value as the second. This comes in handy if tests fail since the error message usually says something like "Expected &lt;thing&gt; but got &lt;other thing&gt;".



##########
connect/runtime/src/main/java/org/apache/kafka/connect/runtime/rest/RestClient.java:
##########
@@ -143,15 +156,12 @@ public static <T> HttpResponse<T> httpRequest(String url, String method, HttpHea
         } catch (IOException | InterruptedException | TimeoutException | ExecutionException e) {
             log.error("IO error forwarding REST request: ", e);
             throw new ConnectRestException(Response.Status.INTERNAL_SERVER_ERROR, "IO Error trying to forward REST request: " + e.getMessage(), e);
+        } catch (ConnectRestException e) {

Review Comment:
   I can see why we're catching and re-throwing this kind of exception here, and I'm also having trouble thinking of anything significantly cleaner, so this might be good enough as-is. But do you think a brief comment might help other developers understand the intent here? The control flow is a little tricky.



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);
+    }
+
+    @BeforeEach
+    public void mockSetup() {
+        httpClient = niceMock(HttpClient.class);
+    }
+
+    @Test
+    public void testSuccess() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.OK.getStatusCode();
+        String expectedResponse = toJsonString(new TestDTO("someContent"));
+        setupHttpClient(statusCode, expectedResponse);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertEquals(toJsonString(httpResp.body()), expectedResponse);
+    }
+
+    @Test
+    public void testNoContent() throws ExecutionException, InterruptedException, TimeoutException {
+        int statusCode = Response.Status.NO_CONTENT.getStatusCode();
+        setupHttpClient(statusCode, null);
+
+        RestClient.HttpResponse<TestDTO> httpResp = httpRequest(httpClient, TEST_TYPE);
+        assertEquals(httpResp.status(), statusCode);
+        assertNull(httpResp.body());
+    }

Review Comment:
   Good call! Nice to have coverage for this case 👍 



##########
connect/runtime/src/test/java/org/apache/kafka/connect/runtime/rest/RestClientTest.java:
##########
@@ -0,0 +1,128 @@
+/*
+ * 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.kafka.connect.runtime.rest;
+
+import com.fasterxml.jackson.annotation.JsonCreator;
+import com.fasterxml.jackson.annotation.JsonProperty;
+import com.fasterxml.jackson.core.JsonProcessingException;
+import com.fasterxml.jackson.core.type.TypeReference;
+import com.fasterxml.jackson.databind.ObjectMapper;
+import org.apache.kafka.connect.runtime.WorkerConfig;
+import org.apache.kafka.connect.runtime.rest.entities.ErrorMessage;
+import org.apache.kafka.connect.runtime.rest.errors.ConnectRestException;
+import org.eclipse.jetty.client.HttpClient;
+import org.eclipse.jetty.client.api.ContentResponse;
+import org.eclipse.jetty.client.api.Request;
+import org.junit.jupiter.api.BeforeEach;
+import org.junit.jupiter.api.Test;
+
+import javax.ws.rs.core.Response;
+import java.util.Collections;
+import java.util.concurrent.ExecutionException;
+import java.util.concurrent.TimeoutException;
+
+import static org.easymock.EasyMock.anyString;
+import static org.easymock.EasyMock.expect;
+import static org.easymock.EasyMock.niceMock;
+import static org.easymock.EasyMock.replay;
+import static org.junit.jupiter.api.Assertions.assertEquals;
+import static org.junit.jupiter.api.Assertions.assertNull;
+import static org.junit.jupiter.api.Assertions.assertThrows;
+
+public class RestClientTest {
+
+    private static final ObjectMapper OBJECT_MAPPER = new ObjectMapper();
+    private static final TypeReference<TestDTO> TEST_TYPE = new TypeReference<TestDTO>() {
+    };
+    private HttpClient httpClient;
+
+    private static String toJsonString(Object obj) {
+        try {
+            return OBJECT_MAPPER.writeValueAsString(obj);
+        } catch (JsonProcessingException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    private static <T> RestClient.HttpResponse<T> httpRequest(HttpClient httpClient, TypeReference<T> typeReference) {
+        return RestClient.httpRequest(
+                httpClient, null, null, null, null, typeReference, null, null);

Review Comment:
   Also, is there anything preventing us from giving a real object for the `requestBodyData` parameter? We already have JSON serialization/deserialization logic set up for the `TestDTO` object, it should be easy enough to take advantage of it.



-- 
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: jira-unsubscribe@kafka.apache.org

For queries about this service, please contact Infrastructure at:
users@infra.apache.org