This is an automated email from the ASF dual-hosted git repository.

chia7712 pushed a commit to branch trunk
in repository https://gitbox.apache.org/repos/asf/kafka.git


The following commit(s) were added to refs/heads/trunk by this push:
     new 3b6c8385cae MINOR: Move client test utilities to test fixtures and 
resolve shadow jar conflicts (#22201)
3b6c8385cae is described below

commit 3b6c8385caedc8c4b5d621682050a13581c827c8
Author: Igor Soarez <[email protected]>
AuthorDate: Tue May 5 23:29:35 2026 +0100

    MINOR: Move client test utilities to test fixtures and resolve shadow jar 
conflicts (#22201)
    
    This change moves TestSslUtils, TestUtils and accessory classes from the
    test classpath in clients to test fixtures, allowing other modules to
    use these classes for testing without having to include all of the test
    classpath for clients.
    
    For further details on the context and purpose of this change, check
    these threads:
    https://github.com/apache/kafka/pull/20376#discussion_r3167281569
    https://github.com/apache/kafka/pull/22193#discussion_r3178891141
    
    Reviewers: Chia-Ping Tsai <[email protected]>
---
 build.gradle                                       | 129 +++++++++++++++------
 checkstyle/import-control.xml                      |   1 +
 .../kafka/clients/admin/KafkaAdminClientTest.java  |  46 --------
 .../kafka/common/protocol/SendBuilderTest.java     |  10 +-
 .../kafka/common/requests/EnvelopeRequestTest.java |   3 +-
 .../common/requests/EnvelopeResponseTest.java      |   3 +-
 .../java/org/apache/kafka/clients/MockClient.java  |   0
 .../kafka/clients/admin/AdminClientTestUtils.java  |   0
 .../clients/admin/AdminClientUnitTestEnv.java      |   0
 .../org/apache/kafka/clients/admin/ConfigTest.java |   0
 .../FailureInjectingTimeoutProcessorFactory.java   |  64 ++++++++++
 .../kafka/clients/admin/FakeForwardingAdmin.java   |   0
 .../admin/KafkaAdminClientInternalFactory.java}    |   9 +-
 .../kafka/clients/admin/MockAdminClient.java       |   0
 .../consumer/internals/MockRebalanceListener.java  |   0
 .../config/provider/MockFileConfigProvider.java    |   0
 .../kafka/common/metrics/FakeMetricsReporter.java  |   0
 .../apache/kafka/common/network/CertStores.java    |   0
 .../kafka/common/network/NetworkTestUtils.java     |   0
 .../apache/kafka/common/network/NioEchoServer.java |   0
 .../record/internal/ArbitraryMemoryRecords.java    |   0
 .../internal/InvalidMemoryRecordsProvider.java     |   0
 .../kafka/common/requests/ByteBufferChannel.java   |  19 +++
 .../kafka/common/requests/RequestTestUtils.java    |   0
 .../kafka/common/security/TestSecurityConfig.java  |   0
 .../authenticator/TestDigestLoginModule.java       |   0
 .../security/authenticator/TestJaasConfig.java     |   0
 .../kafka/common/utils/LogCaptureAppender.java     |   0
 .../org/apache/kafka/common/utils/MockTime.java    |   0
 .../utils/annotation/ApiKeyVersionsProvider.java   |   0
 .../utils/annotation/ApiKeyVersionsSource.java     |   0
 .../apache/kafka/test/MockConsumerInterceptor.java |   0
 .../org/apache/kafka/test/MockDeserializer.java    |   0
 .../org/apache/kafka/test/MockMetricsReporter.java |   0
 .../apache/kafka/test/MockProducerInterceptor.java |   0
 .../java/org/apache/kafka/test/MockSerializer.java |   0
 .../org/apache/kafka/test/NoRetryException.java    |   0
 .../java/org/apache/kafka/test/TestCondition.java  |   0
 .../java/org/apache/kafka/test/TestSslUtils.java   |   0
 .../java/org/apache/kafka/test/TestUtils.java      |  18 ---
 .../org/apache/kafka/test/ValuelessCallable.java   |   0
 .../kafka/api/PlaintextAdminIntegrationTest.scala  |   4 +-
 .../unit/kafka/network/RequestChannelTest.scala    |   3 +-
 .../apache/kafka/snapshot/FileRawSnapshotTest.java |   9 +-
 .../streams/integration/ForeignKeyJoinSuite.java   |   4 -
 45 files changed, 197 insertions(+), 125 deletions(-)

diff --git a/build.gradle b/build.gradle
index e4640eb35fc..3826591e530 100644
--- a/build.gradle
+++ b/build.gradle
@@ -14,6 +14,7 @@
 // limitations under the License.
 
 import org.ajoberstar.grgit.Grgit
+import org.gradle.api.artifacts.FileCollectionDependency
 import java.nio.charset.StandardCharsets
 
 buildscript {
@@ -313,6 +314,24 @@ subprojects {
   task allDepInsight(type: DependencyInsightReportTask) {showingAllVariants = 
false} doLast {}
 
   apply plugin: 'java-library'
+
+  // Resolve conflict between testFixtures and shadow in clients:
+  // testFixtures(project(':clients)) depends on the raw main classpath for 
clients (apiElements) - see "Rewire
+  // test fixtures dependencies to avoid depending on the shadow JAR" in the 
clients project - and this creates
+  // a conflicting variant with the shadow jar. e.g. main classpath for 
streams depends the clients shadow jar,
+  // and the test classpath for streams depends on the client test fixtures, 
which transitively depends on
+  // clients raw main classpath (apiElements) conflicting with the shadow jar. 
To resolve this, we configure
+  // a resolution strategy for every other module that prefers the shadow jar. 
As an alternative to this general
+  // rule, each module that includes the test fixtures dependency could 
exclude the conflicting variant:
+  // testImplementation(testFixtures(project(':clients'))) { exclude group: 
'org.apache.kafka', module: 'clients' }
+  if (project.name != 'clients') {
+    configurations.configureEach {
+      
resolutionStrategy.capabilitiesResolution.withCapability('org.apache.kafka:clients')
 {
+        select(candidates.find { 
it.variantName.toLowerCase().contains('shadow') })
+      }
+    }
+  }
+
   apply plugin: 'checkstyle'
   apply plugin: "com.github.spotbugs"
 
@@ -998,7 +1017,7 @@ project(':server') {
     implementation libs.slf4jApi
     implementation log4j2Libs
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
 
     testImplementation libs.mockitoCore
     testImplementation libs.junitJupiter
@@ -1119,7 +1138,7 @@ project(':core') {
     implementation libs.slf4jApi
     implementation libs.re2j
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':group-coordinator').sourceSets.test.output
     testImplementation project(':share-coordinator').sourceSets.test.output
     testImplementation project(':metadata').sourceSets.test.output
@@ -1397,7 +1416,7 @@ project(':metadata') {
     testImplementation libs.junitJupiter
     testImplementation libs.jqwik
     testImplementation libs.mockitoCore
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':raft').sourceSets.test.output
     testImplementation project(':server-common').sourceSets.test.output
     testImplementation project(':test-common:test-common-util')
@@ -1520,7 +1539,7 @@ project(':group-coordinator') {
     implementation libs.slf4jApi
     implementation libs.hash4j
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':server-common').sourceSets.test.output
     testImplementation project(':coordinator-common').sourceSets.test.output
     testImplementation libs.jacksonDataformatYaml
@@ -1683,7 +1702,7 @@ project(':transaction-coordinator') {
     testImplementation testLog4j2Libs
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':test-common:test-common-runtime')
     testImplementation project(':test-common:test-common-internal-api')
 
@@ -1747,7 +1766,7 @@ project(':coordinator-common') {
     implementation libs.metrics
     implementation libs.hdrHistogram
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':server-common').sourceSets.test.output
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
@@ -1783,7 +1802,7 @@ project(':share-coordinator') {
     implementation libs.metrics
     implementation libs.slf4jApi
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':server-common').sourceSets.test.output
     testImplementation project(':coordinator-common').sourceSets.test.output
     testImplementation libs.junitJupiter
@@ -1879,11 +1898,35 @@ project(':clients') {
     archivesName = "kafka-clients"
   }
 
+  apply plugin: 'java-test-fixtures'
+
   configurations {
     generator
     shadowed
   }
 
+  // Rewire test fixtures dependencies to avoid depending on the shadow JAR:
+  // java-test-fixtures creates dependencies testImplementation -> 
testFixturesApi -> (main artifact)
+  // It prefers the main artifact over the raw main classpath so that 
dependents on fixtures get the same
+  // packaged artifact as production consumers. In this module the main 
artifact is the shadow JAR,
+  // so this creates a test dependency on createVersionFile breaking 
AppInfoParserTest, and worse:
+  // test code is compiled against the original packages, but the shadow JAR 
contains relocated bytecode,
+  // causing runtime errors for various tests. To fix this, we rewire the test 
fixtures dependency to the raw
+  // source set output, so tests continue to use compiled classes and original 
dependencies instead of the shadow JAR.
+  // 
https://github.com/gradle/gradle/blob/v9.4.1/platforms/jvm/plugins-jvm-test-fixtures/src/main/java/org/gradle/api/plugins/JavaTestFixturesPlugin.java#L89-L90
+  afterEvaluate {
+    configurations.testFixturesApi.dependencies.removeIf { dep ->
+      dep instanceof ProjectDependency && dep.name == project.name
+    }
+    dependencies {
+      testFixturesApi files(sourceSets.main.output.classesDirs)
+      testFixturesApi files(sourceSets.main.output.resourcesDir)
+    }
+    tasks.named('compileTestFixturesJava') {
+      dependsOn tasks.named('classes')
+    }
+  }
+
   dependencies {
     implementation libs.zstd
     implementation libs.lz4
@@ -1902,17 +1945,17 @@ project(':clients') {
     compileOnly libs.jacksonJDK8Datatypes
     compileOnly libs.jose4j          // for SASL/OAUTHBEARER JWT validation; 
only used by broker
 
+    testFixturesApi project(':test-common:test-common-util')
+    testFixturesApi libs.bcpkix
+    testFixturesApi libs.junitJupiter
+    testFixturesApi libs.jqwik
+    testFixturesApi testLog4j2Libs
 
-    testImplementation project(':test-common:test-common-util')
-    testImplementation libs.bcpkix
     testImplementation libs.jacksonJakartarsJsonProvider
     testImplementation libs.jose4j
-    testImplementation libs.junitJupiter
-    testImplementation libs.jqwik
     testImplementation libs.spotbugs
     testImplementation libs.mockitoCore
     testImplementation libs.mockitoJunitJupiter // supports MockitoExtension
-    testImplementation testLog4j2Libs
     testImplementation libs.testcontainersKeycloak
     testImplementation libs.testcontainersJunitJupiter
 
@@ -2081,7 +2124,7 @@ project(':clients:clients-integration-tests') {
     testImplementation project(':server')
     testImplementation project(':storage')
     testImplementation project(':core').sourceSets.test.output
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     implementation project(':server-common')
     testImplementation project(':server-common').sourceSets.test.output
     testImplementation project(':metadata')
@@ -2121,7 +2164,7 @@ project(':raft') {
     testImplementation project(':server-common')
     testImplementation project(':server-common').sourceSets.test.output
     testImplementation project(':clients')
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation libs.jacksonDataformatYaml
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
@@ -2218,7 +2261,7 @@ project(':server-common') {
     implementation libs.slf4jApi
 
     testImplementation project(':clients')
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation libs.jacksonDataformatYaml
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
@@ -2278,7 +2321,7 @@ project(':storage:storage-api') {
     implementation libs.slf4jApi
 
     testImplementation project(':clients')
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
     testImplementation testLog4j2Libs
@@ -2347,7 +2390,7 @@ project(':storage') {
     implementation libs.metrics
 
     testImplementation project(':clients')
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':core')
     testImplementation project(':core').sourceSets.test.output
     testImplementation project(':storage:storage-api').sourceSets.test.output
@@ -2540,7 +2583,7 @@ project(':tools') {
     compileOnly libs.spotbugs
 
     testImplementation project(':clients')
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':server')
     testImplementation project(':server').sourceSets.test.output
     testImplementation project(':core')
@@ -2625,7 +2668,7 @@ project(':trogdor') {
     implementation project(':group-coordinator:group-coordinator-api')
 
     testImplementation project(':clients')
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':group-coordinator')
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
@@ -2674,7 +2717,6 @@ project(':shell') {
     implementation libs.jacksonJakartarsJsonProvider
 
     testImplementation project(':clients')
-    testImplementation project(':clients').sourceSets.test.output
     testImplementation project(':core')
     testImplementation project(':server-common')
     testImplementation project(':server-common').sourceSets.test.output
@@ -2726,7 +2768,7 @@ project(':streams') {
     testCompileOnly project(':streams:test-utils')
     testCompileOnly libs.bndlib
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation libs.jacksonDataformatYaml
     testImplementation libs.junitJupiter
     testImplementation libs.bcpkix
@@ -2869,7 +2911,7 @@ project(':streams:streams-scala') {
 
     api libs.scalaLibrary
     testImplementation project(':streams').sourceSets.test.output
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':streams:test-utils')
 
     testImplementation libs.jacksonDataformatYaml
@@ -2921,7 +2963,7 @@ project(':streams:integration-tests') {
     implementation libs.slf4jApi
     implementation libs.scalaLibrary
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':group-coordinator')
     testImplementation project(':server')
     testImplementation project(':server-common')
@@ -2975,7 +3017,7 @@ project(':streams:test-utils') {
 
     implementation libs.slf4jApi
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation libs.jacksonDataformatYaml
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
@@ -3011,7 +3053,7 @@ project(':streams:examples') {
     implementation libs.jacksonAnnotations
 
     testImplementation project(':streams:test-utils')
-    testImplementation project(':clients').sourceSets.test.output // for 
org.apache.kafka.test.IntegrationTest
+    testImplementation testFixtures(project(':clients'))
     testImplementation libs.junitJupiter
     testImplementation libs.hamcrest
     testImplementation testLog4j2Libs
@@ -3450,7 +3492,7 @@ project(':jmh-benchmarks') {
     implementation project(':connect:api')
     implementation project(':connect:transforms')
     implementation project(':connect:json')
-    implementation project(':clients').sourceSets.test.output
+    implementation testFixtures(project(':clients'))
     implementation project(':server-common').sourceSets.test.output
     implementation project(':server').sourceSets.test.output
     implementation project(':metadata').sourceSets.test.output
@@ -3508,7 +3550,7 @@ project(':connect:api') {
     implementation libs.slf4jApi
 
     testImplementation libs.junitJupiter
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation testLog4j2Libs
 
     testRuntimeOnly runtimeTestLibs
@@ -3543,7 +3585,7 @@ project(':connect:transforms') {
     implementation libs.slf4jApi
 
     testImplementation libs.junitJupiter
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation testLog4j2Libs
 
     testRuntimeOnly runtimeTestLibs
@@ -3583,7 +3625,7 @@ project(':connect:json') {
 
     testImplementation libs.junitJupiter
     testImplementation testLog4j2Libs
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
 
     testRuntimeOnly runtimeTestLibs
   }
@@ -3657,7 +3699,7 @@ project(':connect:runtime') {
     swagger libs.jakartaServletApi
     swagger libs.jaxrs2Jakarta
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':core')
     testImplementation project(':server')
     testImplementation project(':metadata')
@@ -3785,7 +3827,7 @@ project(':connect:file') {
     testImplementation libs.junitJupiter
     testImplementation libs.mockitoCore
 
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':connect:runtime')
     testImplementation project(':connect:runtime').sourceSets.test.output
     testImplementation project(':core')
@@ -3829,7 +3871,7 @@ project(':connect:basic-auth-extension') {
     testImplementation libs.mockitoCore
     testImplementation libs.junitJupiter
     testImplementation testLog4j2Libs
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
 
     testRuntimeOnly libs.jerseyContainerServlet
     testRuntimeOnly runtimeTestLibs
@@ -3890,7 +3932,7 @@ project(':connect:mirror') {
     testImplementation libs.junitJupiter
     testImplementation libs.bndlib
     testImplementation libs.mockitoCore
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
     testImplementation project(':connect:runtime').sourceSets.test.output
     testImplementation project(':core')
     testImplementation project(':test-common:test-common-runtime')
@@ -3961,7 +4003,7 @@ project(':connect:mirror-client') {
 
     testImplementation testLog4j2Libs
     testImplementation libs.junitJupiter
-    testImplementation project(':clients').sourceSets.test.output
+    testImplementation testFixtures(project(':clients'))
 
     testRuntimeOnly runtimeTestLibs
   }
@@ -4014,6 +4056,25 @@ task aggregatedJavadoc(type: Javadoc) {
 }
 
 gradle.projectsEvaluated {
+  // Validate that no module depends on 
`project(':clients').sourceSets.test.output`
+  allprojects { proj ->
+    proj.configurations.all { config ->
+      config.dependencies.each { dep ->
+        if (dep instanceof FileCollectionDependency) {
+          dep.files.files.each { file ->
+            if 
(file.absolutePath.endsWith('/clients/build/classes/java/test')) {
+              throw new GradleException("""
+ERROR: Dependency on :clients test output is not allowed.
+Found in project ${proj.path} configuration '${config.name}'.
+Use `testFixtures(project(':clients'))` instead of 
`project(':clients').sourceSets.test.output`
+""")
+            }
+          }
+        }
+      }
+    }
+  }
+
   def projectsWithJavadoc = subprojects.findAll { it.javadoc.enabled }
 
   projectsWithJavadoc.each { sp ->
diff --git a/checkstyle/import-control.xml b/checkstyle/import-control.xml
index a9fb1cdc71d..aba76b4f532 100644
--- a/checkstyle/import-control.xml
+++ b/checkstyle/import-control.xml
@@ -481,6 +481,7 @@
     <allow class="org.apache.kafka.common.compress.Compression" 
exact-match="true" />
     <allow pkg="org.apache.kafka.common.message" />
     <allow pkg="org.apache.kafka.common.record" />
+    <allow pkg="org.apache.kafka.common.requests" />
     <allow pkg="org.apache.kafka.raft" />
     <allow pkg="org.apache.kafka.server.common" />
     <allow pkg="org.apache.kafka.test"/>
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java
 
b/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java
index ca1c5e9be30..84297246266 100644
--- 
a/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java
+++ 
b/clients/src/test/java/org/apache/kafka/clients/admin/KafkaAdminClientTest.java
@@ -10959,52 +10959,6 @@ public class KafkaAdminClientTest {
         assertEquals(elements.length, collection.size(), "There are unexpected 
extra elements in the collection.");
     }
 
-    public static KafkaAdminClient createInternal(AdminClientConfig config, 
KafkaAdminClient.TimeoutProcessorFactory timeoutProcessorFactory) {
-        return KafkaAdminClient.createInternal(config, 
timeoutProcessorFactory);
-    }
-
-    public static class FailureInjectingTimeoutProcessorFactory extends 
KafkaAdminClient.TimeoutProcessorFactory {
-
-        private int numTries = 0;
-
-        private int failuresInjected = 0;
-
-        @Override
-        public KafkaAdminClient.TimeoutProcessor create(long now) {
-            return new FailureInjectingTimeoutProcessor(now);
-        }
-
-        synchronized boolean shouldInjectFailure() {
-            numTries++;
-            if (numTries == 1) {
-                failuresInjected++;
-                return true;
-            }
-            return false;
-        }
-
-        public synchronized int failuresInjected() {
-            return failuresInjected;
-        }
-
-        public final class FailureInjectingTimeoutProcessor extends 
KafkaAdminClient.TimeoutProcessor {
-            public FailureInjectingTimeoutProcessor(long now) {
-                super(now);
-            }
-
-            boolean callHasExpired(KafkaAdminClient.Call call) {
-                if ((!call.isInternal()) && shouldInjectFailure()) {
-                    log.debug("Injecting timeout for {}.", call);
-                    return true;
-                } else {
-                    boolean ret = super.callHasExpired(call);
-                    log.debug("callHasExpired({}) = {}", call, ret);
-                    return ret;
-                }
-            }
-        }
-    }
-
     @ParameterizedTest
     @CsvSource({ "false, false", "false, true", "true, false", "true, true" })
     public void testAddRaftVoterRequest(boolean fail, boolean sendClusterId) 
throws Exception {
diff --git 
a/clients/src/test/java/org/apache/kafka/common/protocol/SendBuilderTest.java 
b/clients/src/test/java/org/apache/kafka/common/protocol/SendBuilderTest.java
index d1cd59066b3..40673bd1546 100644
--- 
a/clients/src/test/java/org/apache/kafka/common/protocol/SendBuilderTest.java
+++ 
b/clients/src/test/java/org/apache/kafka/common/protocol/SendBuilderTest.java
@@ -23,8 +23,8 @@ import org.apache.kafka.common.record.internal.MemoryRecords;
 import org.apache.kafka.common.record.internal.MemoryRecordsBuilder;
 import org.apache.kafka.common.record.internal.SimpleRecord;
 import org.apache.kafka.common.record.internal.UnalignedMemoryRecords;
+import org.apache.kafka.common.requests.ByteBufferChannel;
 import org.apache.kafka.common.utils.Utils;
-import org.apache.kafka.test.TestUtils;
 
 import org.junit.jupiter.api.Test;
 
@@ -52,7 +52,7 @@ public class SendBuilderTest {
         zeroCopyBuffer.put(overwrittenData);
         zeroCopyBuffer.rewind();
 
-        ByteBuffer buffer = TestUtils.toBuffer(send);
+        ByteBuffer buffer = ByteBufferChannel.toBuffer(send);
         assertEquals(8 + data.length, buffer.remaining());
         assertEquals(5, buffer.getInt());
         assertEquals("bar", getString(buffer, data.length));
@@ -77,7 +77,7 @@ public class SendBuilderTest {
         assertEquals(2, buffer.position());
 
         Send send = builder.build();
-        ByteBuffer readBuffer = TestUtils.toBuffer(send);
+        ByteBuffer readBuffer = ByteBufferChannel.toBuffer(send);
         assertEquals("yolo", getString(readBuffer, 4));
     }
 
@@ -96,7 +96,7 @@ public class SendBuilderTest {
         buffer.rewind();
         MemoryRecords overwrittenRecords = createRecords(buffer, "bar");
 
-        ByteBuffer readBuffer = TestUtils.toBuffer(send);
+        ByteBuffer readBuffer = ByteBufferChannel.toBuffer(send);
         assertEquals(5, readBuffer.getInt());
         assertEquals(overwrittenRecords, getRecords(readBuffer, 
records.sizeInBytes()));
         assertEquals(15, readBuffer.getInt());
@@ -127,7 +127,7 @@ public class SendBuilderTest {
         buffer.rewind();
         MemoryRecords overwrittenRecords = createRecords(buffer, "bar");
 
-        ByteBuffer readBuffer = TestUtils.toBuffer(send);
+        ByteBuffer readBuffer = ByteBufferChannel.toBuffer(send);
         assertEquals(5, readBuffer.getInt());
         assertEquals(overwrittenRecords, getRecords(readBuffer, 
records.sizeInBytes()));
         assertEquals(15, readBuffer.getInt());
diff --git 
a/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeRequestTest.java
 
b/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeRequestTest.java
index e3250ffa4f2..5cf178b8b51 100644
--- 
a/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeRequestTest.java
+++ 
b/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeRequestTest.java
@@ -22,7 +22,6 @@ import org.apache.kafka.common.protocol.ApiKeys;
 import org.apache.kafka.common.protocol.ByteBufferAccessor;
 import org.apache.kafka.common.security.auth.KafkaPrincipal;
 import 
org.apache.kafka.common.security.authenticator.DefaultKafkaPrincipalBuilder;
-import org.apache.kafka.test.TestUtils;
 
 import org.junit.jupiter.api.Test;
 
@@ -57,7 +56,7 @@ public class EnvelopeRequestTest {
             ).build(version);
 
             Send send = request.toSend(header);
-            ByteBuffer buffer = TestUtils.toBuffer(send);
+            ByteBuffer buffer = ByteBufferChannel.toBuffer(send);
             assertEquals(send.size() - 4, buffer.getInt());
             RequestHeader parsedHeader = RequestHeader.parse(buffer);
             assertEquals(header.size(), parsedHeader.size());
diff --git 
a/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeResponseTest.java
 
b/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeResponseTest.java
index 6afe8319635..a712c5989e3 100644
--- 
a/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeResponseTest.java
+++ 
b/clients/src/test/java/org/apache/kafka/common/requests/EnvelopeResponseTest.java
@@ -21,7 +21,6 @@ import org.apache.kafka.common.network.Send;
 import org.apache.kafka.common.protocol.ApiKeys;
 import org.apache.kafka.common.protocol.ByteBufferAccessor;
 import org.apache.kafka.common.protocol.Errors;
-import org.apache.kafka.test.TestUtils;
 
 import org.junit.jupiter.api.Test;
 
@@ -40,7 +39,7 @@ class EnvelopeResponseTest {
             ResponseHeader header = new ResponseHeader(15, headerVersion);
 
             Send send = response.toSend(header, version);
-            ByteBuffer buffer = TestUtils.toBuffer(send);
+            ByteBuffer buffer = ByteBufferChannel.toBuffer(send);
             assertEquals(send.size() - 4, buffer.getInt());
             ResponseHeader parsedHeader = ResponseHeader.parse(buffer, 
headerVersion);
             assertEquals(header.size(), parsedHeader.size());
diff --git a/clients/src/test/java/org/apache/kafka/clients/MockClient.java 
b/clients/src/testFixtures/java/org/apache/kafka/clients/MockClient.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/clients/MockClient.java
rename to clients/src/testFixtures/java/org/apache/kafka/clients/MockClient.java
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/admin/AdminClientTestUtils.java
 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/AdminClientTestUtils.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/clients/admin/AdminClientTestUtils.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/clients/admin/AdminClientTestUtils.java
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/admin/AdminClientUnitTestEnv.java
 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/AdminClientUnitTestEnv.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/clients/admin/AdminClientUnitTestEnv.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/clients/admin/AdminClientUnitTestEnv.java
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/admin/ConfigTest.java 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/ConfigTest.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/clients/admin/ConfigTest.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/clients/admin/ConfigTest.java
diff --git 
a/clients/src/testFixtures/java/org/apache/kafka/clients/admin/FailureInjectingTimeoutProcessorFactory.java
 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/FailureInjectingTimeoutProcessorFactory.java
new file mode 100644
index 00000000000..61e9341944c
--- /dev/null
+++ 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/FailureInjectingTimeoutProcessorFactory.java
@@ -0,0 +1,64 @@
+/*
+ * 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.clients.admin;
+
+import org.slf4j.Logger;
+import org.slf4j.LoggerFactory;
+
+public class FailureInjectingTimeoutProcessorFactory extends 
KafkaAdminClient.TimeoutProcessorFactory {
+
+    private static final Logger log = 
LoggerFactory.getLogger(FailureInjectingTimeoutProcessorFactory.class);
+
+    private int numTries = 0;
+
+    private int failuresInjected = 0;
+
+    @Override
+    public KafkaAdminClient.TimeoutProcessor create(long now) {
+        return new FailureInjectingTimeoutProcessor(now);
+    }
+
+    synchronized boolean shouldInjectFailure() {
+        numTries++;
+        if (numTries == 1) {
+            failuresInjected++;
+            return true;
+        }
+        return false;
+    }
+
+    public synchronized int failuresInjected() {
+        return failuresInjected;
+    }
+
+    public final class FailureInjectingTimeoutProcessor extends 
KafkaAdminClient.TimeoutProcessor {
+        public FailureInjectingTimeoutProcessor(long now) {
+            super(now);
+        }
+
+        boolean callHasExpired(KafkaAdminClient.Call call) {
+            if ((!call.isInternal()) && shouldInjectFailure()) {
+                log.debug("Injecting timeout for {}.", call);
+                return true;
+            } else {
+                boolean ret = super.callHasExpired(call);
+                log.debug("callHasExpired({}) = {}", call, ret);
+                return ret;
+            }
+        }
+    }
+}
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/admin/FakeForwardingAdmin.java 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/FakeForwardingAdmin.java
similarity index 100%
copy from 
clients/src/test/java/org/apache/kafka/clients/admin/FakeForwardingAdmin.java
copy to 
clients/src/testFixtures/java/org/apache/kafka/clients/admin/FakeForwardingAdmin.java
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/admin/FakeForwardingAdmin.java 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/KafkaAdminClientInternalFactory.java
similarity index 75%
rename from 
clients/src/test/java/org/apache/kafka/clients/admin/FakeForwardingAdmin.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/clients/admin/KafkaAdminClientInternalFactory.java
index 9e081d52026..72d977e2adf 100644
--- 
a/clients/src/test/java/org/apache/kafka/clients/admin/FakeForwardingAdmin.java
+++ 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/KafkaAdminClientInternalFactory.java
@@ -14,13 +14,10 @@
  * See the License for the specific language governing permissions and
  * limitations under the License.
  */
-
 package org.apache.kafka.clients.admin;
 
-import java.util.Map;
-
-public class FakeForwardingAdmin extends ForwardingAdmin {
-    public FakeForwardingAdmin(Map<String, Object> configs) {
-        super(configs);
+public class KafkaAdminClientInternalFactory {
+    public static KafkaAdminClient createInternal(AdminClientConfig config, 
KafkaAdminClient.TimeoutProcessorFactory timeoutProcessorFactory) {
+        return KafkaAdminClient.createInternal(config, 
timeoutProcessorFactory);
     }
 }
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/admin/MockAdminClient.java 
b/clients/src/testFixtures/java/org/apache/kafka/clients/admin/MockAdminClient.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/clients/admin/MockAdminClient.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/clients/admin/MockAdminClient.java
diff --git 
a/clients/src/test/java/org/apache/kafka/clients/consumer/internals/MockRebalanceListener.java
 
b/clients/src/testFixtures/java/org/apache/kafka/clients/consumer/internals/MockRebalanceListener.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/clients/consumer/internals/MockRebalanceListener.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/clients/consumer/internals/MockRebalanceListener.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/config/provider/MockFileConfigProvider.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/config/provider/MockFileConfigProvider.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/config/provider/MockFileConfigProvider.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/config/provider/MockFileConfigProvider.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/metrics/FakeMetricsReporter.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/metrics/FakeMetricsReporter.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/metrics/FakeMetricsReporter.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/metrics/FakeMetricsReporter.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/network/CertStores.java 
b/clients/src/testFixtures/java/org/apache/kafka/common/network/CertStores.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/network/CertStores.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/network/CertStores.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/network/NetworkTestUtils.java 
b/clients/src/testFixtures/java/org/apache/kafka/common/network/NetworkTestUtils.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/network/NetworkTestUtils.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/network/NetworkTestUtils.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/network/NioEchoServer.java 
b/clients/src/testFixtures/java/org/apache/kafka/common/network/NioEchoServer.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/network/NioEchoServer.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/network/NioEchoServer.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/record/internal/ArbitraryMemoryRecords.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/record/internal/ArbitraryMemoryRecords.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/record/internal/ArbitraryMemoryRecords.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/record/internal/ArbitraryMemoryRecords.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/record/internal/InvalidMemoryRecordsProvider.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/record/internal/InvalidMemoryRecordsProvider.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/record/internal/InvalidMemoryRecordsProvider.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/record/internal/InvalidMemoryRecordsProvider.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/requests/ByteBufferChannel.java 
b/clients/src/testFixtures/java/org/apache/kafka/common/requests/ByteBufferChannel.java
similarity index 79%
rename from 
clients/src/test/java/org/apache/kafka/common/requests/ByteBufferChannel.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/requests/ByteBufferChannel.java
index 7370f502a07..f8bdcfde809 100644
--- 
a/clients/src/test/java/org/apache/kafka/common/requests/ByteBufferChannel.java
+++ 
b/clients/src/testFixtures/java/org/apache/kafka/common/requests/ByteBufferChannel.java
@@ -16,12 +16,16 @@
  */
 package org.apache.kafka.common.requests;
 
+import org.apache.kafka.common.network.Send;
 import org.apache.kafka.common.network.TransferableChannel;
+import org.apache.kafka.common.record.internal.UnalignedRecords;
 
 import java.io.IOException;
 import java.nio.ByteBuffer;
 import java.nio.channels.FileChannel;
 
+import static org.junit.jupiter.api.Assertions.assertEquals;
+
 public class ByteBufferChannel implements TransferableChannel {
     private final ByteBuffer buf;
     private boolean closed = false;
@@ -76,4 +80,19 @@ public class ByteBufferChannel implements 
TransferableChannel {
     public long transferFrom(FileChannel fileChannel, long position, long 
count) throws IOException {
         return fileChannel.transferTo(position, count, this);
     }
+
+    public static ByteBuffer toBuffer(Send send) {
+        ByteBufferChannel channel = new ByteBufferChannel(send.size());
+        try {
+            assertEquals(send.size(), send.writeTo(channel));
+            channel.close();
+            return channel.buffer();
+        } catch (IOException e) {
+            throw new RuntimeException(e);
+        }
+    }
+
+    public static ByteBuffer toBuffer(UnalignedRecords records) {
+        return toBuffer(records.toSend());
+    }
 }
diff --git 
a/clients/src/test/java/org/apache/kafka/common/requests/RequestTestUtils.java 
b/clients/src/testFixtures/java/org/apache/kafka/common/requests/RequestTestUtils.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/requests/RequestTestUtils.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/requests/RequestTestUtils.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/security/TestSecurityConfig.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/security/TestSecurityConfig.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/security/TestSecurityConfig.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/security/TestSecurityConfig.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/security/authenticator/TestDigestLoginModule.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/security/authenticator/TestDigestLoginModule.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/security/authenticator/TestDigestLoginModule.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/security/authenticator/TestDigestLoginModule.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/security/authenticator/TestJaasConfig.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/utils/LogCaptureAppender.java 
b/clients/src/testFixtures/java/org/apache/kafka/common/utils/LogCaptureAppender.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/utils/LogCaptureAppender.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/utils/LogCaptureAppender.java
diff --git a/clients/src/test/java/org/apache/kafka/common/utils/MockTime.java 
b/clients/src/testFixtures/java/org/apache/kafka/common/utils/MockTime.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/common/utils/MockTime.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/utils/MockTime.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsProvider.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsProvider.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsProvider.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsProvider.java
diff --git 
a/clients/src/test/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsSource.java
 
b/clients/src/testFixtures/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsSource.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsSource.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/common/utils/annotation/ApiKeyVersionsSource.java
diff --git 
a/clients/src/test/java/org/apache/kafka/test/MockConsumerInterceptor.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/MockConsumerInterceptor.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/test/MockConsumerInterceptor.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/test/MockConsumerInterceptor.java
diff --git a/clients/src/test/java/org/apache/kafka/test/MockDeserializer.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/MockDeserializer.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/test/MockDeserializer.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/test/MockDeserializer.java
diff --git 
a/clients/src/test/java/org/apache/kafka/test/MockMetricsReporter.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/MockMetricsReporter.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/test/MockMetricsReporter.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/test/MockMetricsReporter.java
diff --git 
a/clients/src/test/java/org/apache/kafka/test/MockProducerInterceptor.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/MockProducerInterceptor.java
similarity index 100%
rename from 
clients/src/test/java/org/apache/kafka/test/MockProducerInterceptor.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/test/MockProducerInterceptor.java
diff --git a/clients/src/test/java/org/apache/kafka/test/MockSerializer.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/MockSerializer.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/test/MockSerializer.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/test/MockSerializer.java
diff --git a/clients/src/test/java/org/apache/kafka/test/NoRetryException.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/NoRetryException.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/test/NoRetryException.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/test/NoRetryException.java
diff --git a/clients/src/test/java/org/apache/kafka/test/TestCondition.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/TestCondition.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/test/TestCondition.java
rename to clients/src/testFixtures/java/org/apache/kafka/test/TestCondition.java
diff --git a/clients/src/test/java/org/apache/kafka/test/TestSslUtils.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/TestSslUtils.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/test/TestSslUtils.java
rename to clients/src/testFixtures/java/org/apache/kafka/test/TestSslUtils.java
diff --git a/clients/src/test/java/org/apache/kafka/test/TestUtils.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/TestUtils.java
similarity index 98%
rename from clients/src/test/java/org/apache/kafka/test/TestUtils.java
rename to clients/src/testFixtures/java/org/apache/kafka/test/TestUtils.java
index 871fe764817..5bf8076caf4 100644
--- a/clients/src/test/java/org/apache/kafka/test/TestUtils.java
+++ b/clients/src/testFixtures/java/org/apache/kafka/test/TestUtils.java
@@ -28,12 +28,9 @@ import org.apache.kafka.common.feature.SupportedVersionRange;
 import org.apache.kafka.common.message.ApiMessageType;
 import org.apache.kafka.common.message.ApiVersionsResponseData;
 import org.apache.kafka.common.network.NetworkReceive;
-import org.apache.kafka.common.network.Send;
 import org.apache.kafka.common.protocol.ApiKeys;
 import org.apache.kafka.common.protocol.Errors;
-import org.apache.kafka.common.record.internal.UnalignedRecords;
 import org.apache.kafka.common.requests.ApiVersionsResponse;
-import org.apache.kafka.common.requests.ByteBufferChannel;
 import org.apache.kafka.common.requests.MetadataResponse.PartitionMetadata;
 import org.apache.kafka.common.requests.RequestHeader;
 import org.apache.kafka.common.serialization.StringDeserializer;
@@ -565,21 +562,6 @@ public class TestUtils {
         return list;
     }
 
-    public static ByteBuffer toBuffer(Send send) {
-        ByteBufferChannel channel = new ByteBufferChannel(send.size());
-        try {
-            assertEquals(send.size(), send.writeTo(channel));
-            channel.close();
-            return channel.buffer();
-        } catch (IOException e) {
-            throw new RuntimeException(e);
-        }
-    }
-
-    public static ByteBuffer toBuffer(UnalignedRecords records) {
-        return toBuffer(records.toSend());
-    }
-
     /**
      * Assert that a future raises an expected exception cause type.
      * This method will wait for the future to complete or timeout(15000 
milliseconds).
diff --git a/clients/src/test/java/org/apache/kafka/test/ValuelessCallable.java 
b/clients/src/testFixtures/java/org/apache/kafka/test/ValuelessCallable.java
similarity index 100%
rename from clients/src/test/java/org/apache/kafka/test/ValuelessCallable.java
rename to 
clients/src/testFixtures/java/org/apache/kafka/test/ValuelessCallable.java
diff --git 
a/core/src/test/scala/integration/kafka/api/PlaintextAdminIntegrationTest.scala 
b/core/src/test/scala/integration/kafka/api/PlaintextAdminIntegrationTest.scala
index 8759fa38128..bcc2f8d4817 100644
--- 
a/core/src/test/scala/integration/kafka/api/PlaintextAdminIntegrationTest.scala
+++ 
b/core/src/test/scala/integration/kafka/api/PlaintextAdminIntegrationTest.scala
@@ -1862,8 +1862,8 @@ class PlaintextAdminIntegrationTest extends 
BaseAdminIntegrationTest {
     val config = createConfig
     config.put(AdminClientConfig.DEFAULT_API_TIMEOUT_MS_CONFIG, "100000000")
     config.put(AdminClientConfig.RETRIES_CONFIG, "0")
-    val factory = new 
KafkaAdminClientTest.FailureInjectingTimeoutProcessorFactory()
-    client = KafkaAdminClientTest.createInternal(new 
AdminClientConfig(config), factory)
+    val factory = new FailureInjectingTimeoutProcessorFactory()
+    client = KafkaAdminClientInternalFactory.createInternal(new 
AdminClientConfig(config), factory)
     val future = client.createTopics(Seq("mytopic", "mytopic2").map(new 
NewTopic(_, 1, 1.toShort)).asJava,
         new CreateTopicsOptions().validateOnly(true)).all()
     assertFutureThrows(classOf[TimeoutException], future)
diff --git a/core/src/test/scala/unit/kafka/network/RequestChannelTest.scala 
b/core/src/test/scala/unit/kafka/network/RequestChannelTest.scala
index 800e6c2039d..582215500b7 100644
--- a/core/src/test/scala/unit/kafka/network/RequestChannelTest.scala
+++ b/core/src/test/scala/unit/kafka/network/RequestChannelTest.scala
@@ -36,7 +36,6 @@ import org.apache.kafka.common.utils.Utils
 import org.apache.kafka.common.utils.internals.SecurityUtils
 import org.apache.kafka.network.RequestConvertToJson
 import org.apache.kafka.network.metrics.RequestChannelMetrics
-import org.apache.kafka.test
 import org.junit.jupiter.api.Assertions._
 import org.junit.jupiter.api._
 import org.junit.jupiter.params.ParameterizedTest
@@ -368,7 +367,7 @@ class RequestChannelTest {
     val envelope = unwrapped.envelope.get
 
     val send = unwrapped.buildResponseSend(response)
-    val sendBytes = test.TestUtils.toBuffer(send)
+    val sendBytes = ByteBufferChannel.toBuffer(send)
 
     // We need to read the size field before `parseResponse` below
     val size = sendBytes.getInt
diff --git 
a/raft/src/test/java/org/apache/kafka/snapshot/FileRawSnapshotTest.java 
b/raft/src/test/java/org/apache/kafka/snapshot/FileRawSnapshotTest.java
index 73d88d1d09f..32d5d891c4b 100644
--- a/raft/src/test/java/org/apache/kafka/snapshot/FileRawSnapshotTest.java
+++ b/raft/src/test/java/org/apache/kafka/snapshot/FileRawSnapshotTest.java
@@ -23,6 +23,7 @@ import org.apache.kafka.common.record.internal.RecordBatch;
 import org.apache.kafka.common.record.internal.SimpleRecord;
 import org.apache.kafka.common.record.internal.UnalignedFileRecords;
 import org.apache.kafka.common.record.internal.UnalignedMemoryRecords;
+import org.apache.kafka.common.requests.ByteBufferChannel;
 import org.apache.kafka.common.utils.Utils;
 import 
org.apache.kafka.common.utils.internals.BufferSupplier.GrowableBufferSupplier;
 import org.apache.kafka.server.common.OffsetAndEpoch;
@@ -157,12 +158,12 @@ public final class FileRawSnapshotTest {
             UnalignedFileRecords record1 = (UnalignedFileRecords) 
snapshot.slice(0, totalSize / 2);
             UnalignedFileRecords record2 = (UnalignedFileRecords) 
snapshot.slice(totalSize / 2, totalSize - totalSize / 2);
 
-            assertEquals(buffer1, TestUtils.toBuffer(record1));
-            assertEquals(buffer2, TestUtils.toBuffer(record2));
+            assertEquals(buffer1, ByteBufferChannel.toBuffer(record1));
+            assertEquals(buffer2, ByteBufferChannel.toBuffer(record2));
 
             ByteBuffer readBuffer = ByteBuffer.allocate(record1.sizeInBytes() 
+ record2.sizeInBytes());
-            readBuffer.put(TestUtils.toBuffer(record1));
-            readBuffer.put(TestUtils.toBuffer(record2));
+            readBuffer.put(ByteBufferChannel.toBuffer(record1));
+            readBuffer.put(ByteBufferChannel.toBuffer(record2));
             readBuffer.flip();
             assertEquals(expectedBuffer, readBuffer);
         }
diff --git 
a/streams/integration-tests/src/test/java/org/apache/kafka/streams/integration/ForeignKeyJoinSuite.java
 
b/streams/integration-tests/src/test/java/org/apache/kafka/streams/integration/ForeignKeyJoinSuite.java
index 629cc1229f0..ca64869a3af 100644
--- 
a/streams/integration-tests/src/test/java/org/apache/kafka/streams/integration/ForeignKeyJoinSuite.java
+++ 
b/streams/integration-tests/src/test/java/org/apache/kafka/streams/integration/ForeignKeyJoinSuite.java
@@ -16,8 +16,6 @@
  */
 package org.apache.kafka.streams.integration;
 
-import org.apache.kafka.common.utils.BytesTest;
-import org.apache.kafka.common.utils.internals.ByteUtilsTest;
 import 
org.apache.kafka.streams.kstream.internals.KTableKTableForeignKeyJoinScenarioTest;
 import 
org.apache.kafka.streams.kstream.internals.foreignkeyjoin.CombinedKeySchemaTest;
 import 
org.apache.kafka.streams.kstream.internals.foreignkeyjoin.ResponseJoinProcessorSupplierTest;
@@ -38,8 +36,6 @@ import org.junit.platform.suite.api.Suite;
  */
 @Suite
 @SelectClasses({
-    BytesTest.class,
-    ByteUtilsTest.class,
     KTableKTableForeignKeyInnerJoinMultiIntegrationTest.class,
     KTableKTableForeignKeyJoinIntegrationTest.class,
     KTableKTableForeignKeyJoinMaterializationIntegrationTest.class,

Reply via email to