Skip to content

Commit dcdb4a0

Browse files
committed
MINOR: exclude all src/generated and src/generated-test (#10671)
Reviewers: Anna Sophie Blee-Goldman <[email protected]>
1 parent cb86a0d commit dcdb4a0

File tree

65 files changed

+432
-86
lines changed

Some content is hidden

Large Commits have some content hidden by default. Use the searchbox below for content that may be hidden.

65 files changed

+432
-86
lines changed

build.gradle

Lines changed: 62 additions & 6 deletions
Original file line numberDiff line numberDiff line change
@@ -779,6 +779,7 @@ project(':core') {
779779
api project(':clients')
780780
api libs.scalaLibrary
781781

782+
implementation project(':server-common')
782783
implementation project(':metadata')
783784
implementation project(':raft')
784785
implementation project(':storage')
@@ -1003,10 +1004,6 @@ project(':core') {
10031004
from(project(':connect:mirror').configurations.runtimeClasspath) { into("libs/") }
10041005
from(project(':connect:mirror-client').jar) { into("libs/") }
10051006
from(project(':connect:mirror-client').configurations.runtimeClasspath) { into("libs/") }
1006-
from(project(':storage').jar) { into("libs/") }
1007-
from(project(':storage').configurations.runtimeClasspath) { into("libs/") }
1008-
from(project(':storage:api').jar) { into("libs/") }
1009-
from(project(':storage:api').configurations.runtimeClasspath) { into("libs/") }
10101007
from(project(':streams').jar) { into("libs/") }
10111008
from(project(':streams').configurations.runtimeClasspath) { into("libs/") }
10121009
from(project(':streams:streams-scala').jar) { into("libs/") }
@@ -1070,6 +1067,7 @@ project(':metadata') {
10701067
archivesBaseName = "kafka-metadata"
10711068

10721069
dependencies {
1070+
implementation project(':server-common')
10731071
implementation project(':clients')
10741072
implementation libs.jacksonDatabind
10751073
implementation libs.jacksonJDK8Datatypes
@@ -1267,11 +1265,13 @@ project(':raft') {
12671265
archivesBaseName = "kafka-raft"
12681266

12691267
dependencies {
1268+
implementation project(':server-common')
12701269
implementation project(':clients')
12711270
implementation project(':metadata')
12721271
implementation libs.slf4jApi
12731272
implementation libs.jacksonDatabind
12741273

1274+
testImplementation project(':server-common')
12751275
testImplementation project(':clients')
12761276
testImplementation project(':clients').sourceSets.test.output
12771277
testImplementation libs.junitJupiter
@@ -1345,6 +1345,62 @@ project(':raft') {
13451345
}
13461346
}
13471347

1348+
project(':server-common') {
1349+
archivesBaseName = "kafka-server-common"
1350+
1351+
dependencies {
1352+
api project(':clients')
1353+
implementation libs.slf4jApi
1354+
1355+
testImplementation project(':clients')
1356+
testImplementation project(':clients').sourceSets.test.output
1357+
testImplementation libs.junitJupiter
1358+
testImplementation libs.mockitoCore
1359+
1360+
testRuntimeOnly libs.slf4jlog4j
1361+
}
1362+
1363+
task createVersionFile(dependsOn: determineCommitId) {
1364+
ext.receiptFile = file("$buildDir/kafka/$buildVersionFileName")
1365+
outputs.file receiptFile
1366+
outputs.upToDateWhen { false }
1367+
doLast {
1368+
def data = [
1369+
commitId: commitId,
1370+
version: version,
1371+
]
1372+
1373+
receiptFile.parentFile.mkdirs()
1374+
def content = data.entrySet().collect { "$it.key=$it.value" }.sort().join("\n")
1375+
receiptFile.setText(content, "ISO-8859-1")
1376+
}
1377+
}
1378+
1379+
sourceSets {
1380+
main {
1381+
java {
1382+
srcDirs = ["src/main/java"]
1383+
}
1384+
}
1385+
test {
1386+
java {
1387+
srcDirs = ["src/test/java"]
1388+
}
1389+
}
1390+
}
1391+
1392+
jar {
1393+
dependsOn createVersionFile
1394+
from("$buildDir") {
1395+
include "kafka/$buildVersionFileName"
1396+
}
1397+
}
1398+
1399+
clean.doFirst {
1400+
delete "$buildDir/kafka/"
1401+
}
1402+
}
1403+
13481404
project(':storage:api') {
13491405
archivesBaseName = "kafka-storage-api"
13501406

@@ -1410,9 +1466,8 @@ project(':storage') {
14101466

14111467
dependencies {
14121468
implementation project(':storage:api')
1469+
implementation project(':server-common')
14131470
implementation project(':clients')
1414-
implementation project(':metadata')
1415-
implementation project(':raft')
14161471
implementation libs.slf4jApi
14171472
implementation libs.jacksonDatabind
14181473

@@ -1585,6 +1640,7 @@ project(':shell') {
15851640
implementation libs.jacksonJDK8Datatypes
15861641
implementation libs.jline
15871642
implementation libs.slf4jApi
1643+
implementation project(':server-common')
15881644
implementation project(':clients')
15891645
implementation project(':core')
15901646
implementation project(':log4j-appender')

checkstyle/import-control-core.xml

Lines changed: 1 addition & 0 deletions
Original file line numberDiff line numberDiff line change
@@ -54,6 +54,7 @@
5454
<allow pkg="org.apache.kafka.test"/>
5555
<allow pkg="org.apache.kafka.metadata" />
5656
<allow pkg="org.apache.kafka.metalog" />
57+
<allow pkg="org.apache.kafka.server.common" />
5758
</subpackage>
5859

5960
<subpackage name="tools">

checkstyle/import-control.xml

Lines changed: 14 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -221,6 +221,7 @@
221221
<allow pkg="org.apache.kafka.metadata" />
222222
<allow pkg="org.apache.kafka.metalog" />
223223
<allow pkg="org.apache.kafka.queue" />
224+
<allow pkg="org.apache.kafka.server.common" />
224225
<allow pkg="org.apache.kafka.test" />
225226
<allow pkg="org.apache.kafka.timeline" />
226227
</subpackage>
@@ -231,6 +232,7 @@
231232
<allow pkg="org.apache.kafka.common.message" />
232233
<allow pkg="org.apache.kafka.common.metadata" />
233234
<allow pkg="org.apache.kafka.common.protocol" />
235+
<allow pkg="org.apache.kafka.server.common" />
234236
<allow pkg="org.apache.kafka.test" />
235237
</subpackage>
236238

@@ -240,6 +242,7 @@
240242
<allow pkg="org.apache.kafka.metadata" />
241243
<allow pkg="org.apache.kafka.metalog" />
242244
<allow pkg="org.apache.kafka.queue" />
245+
<allow pkg="org.apache.kafka.server.common" />
243246
<allow pkg="org.apache.kafka.test" />
244247
</subpackage>
245248

@@ -266,13 +269,16 @@
266269

267270
<subpackage name="server">
268271
<allow pkg="org.apache.kafka.common" />
269-
<allow pkg="org.apache.kafka.test" />
270-
<allow pkg="com.fasterxml.jackson" />
272+
273+
<subpackage name="common">
274+
<allow pkg="org.apache.kafka.server.common" />
275+
</subpackage>
271276

272277
<subpackage name="log">
278+
<allow pkg="com.fasterxml.jackson" />
279+
<allow pkg="org.apache.kafka.server.common" />
273280
<allow pkg="org.apache.kafka.server.log" />
274-
<allow pkg="org.apache.kafka.metadata" />
275-
<allow pkg="org.apache.kafka.raft.metadata" />
281+
<allow pkg="org.apache.kafka.test" />
276282
</subpackage>
277283
</subpackage>
278284

@@ -287,6 +293,7 @@
287293
<allow pkg="org.apache.kafka.metalog"/>
288294
<allow pkg="org.apache.kafka.queue"/>
289295
<allow pkg="org.apache.kafka.raft"/>
296+
<allow pkg="org.apache.kafka.server.common" />
290297
<allow pkg="org.apache.kafka.shell"/>
291298
<allow pkg="org.apache.kafka.snapshot"/>
292299
<allow pkg="org.jline"/>
@@ -410,6 +417,8 @@
410417
<allow pkg="org.apache.kafka.common.requests" />
411418
<allow pkg="org.apache.kafka.common.protocol" />
412419
<allow pkg="org.apache.kafka.metalog" />
420+
<allow pkg="org.apache.kafka.server.common" />
421+
<allow pkg="org.apache.kafka.server.common.serialization" />
413422
<allow pkg="org.apache.kafka.test"/>
414423
<allow pkg="com.fasterxml.jackson" />
415424
<allow pkg="net.jqwik"/>
@@ -418,6 +427,7 @@
418427
<subpackage name="snapshot">
419428
<allow pkg="org.apache.kafka.common.record" />
420429
<allow pkg="org.apache.kafka.raft" />
430+
<allow pkg="org.apache.kafka.server.common" />
421431
<allow pkg="org.apache.kafka.test"/>
422432
</subpackage>
423433

core/src/main/scala/kafka/coordinator/transaction/TransactionMetadata.scala

Lines changed: 11 additions & 15 deletions
Original file line numberDiff line numberDiff line change
@@ -64,6 +64,8 @@ private[transaction] sealed trait TransactionState {
6464
* Get the name of this state. This is exposed through the `DescribeTransactions` API.
6565
*/
6666
def name: String
67+
68+
def validPreviousStates: Set[TransactionState]
6769
}
6870

6971
/**
@@ -75,6 +77,7 @@ private[transaction] sealed trait TransactionState {
7577
private[transaction] case object Empty extends TransactionState {
7678
val id: Byte = 0
7779
val name: String = "Empty"
80+
val validPreviousStates: Set[TransactionState] = Set(Empty, CompleteCommit, CompleteAbort)
7881
}
7982

8083
/**
@@ -88,6 +91,7 @@ private[transaction] case object Empty extends TransactionState {
8891
private[transaction] case object Ongoing extends TransactionState {
8992
val id: Byte = 1
9093
val name: String = "Ongoing"
94+
val validPreviousStates: Set[TransactionState] = Set(Ongoing, Empty, CompleteCommit, CompleteAbort)
9195
}
9296

9397
/**
@@ -98,6 +102,7 @@ private[transaction] case object Ongoing extends TransactionState {
98102
private[transaction] case object PrepareCommit extends TransactionState {
99103
val id: Byte = 2
100104
val name: String = "PrepareCommit"
105+
val validPreviousStates: Set[TransactionState] = Set(Ongoing)
101106
}
102107

103108
/**
@@ -108,6 +113,7 @@ private[transaction] case object PrepareCommit extends TransactionState {
108113
private[transaction] case object PrepareAbort extends TransactionState {
109114
val id: Byte = 3
110115
val name: String = "PrepareAbort"
116+
val validPreviousStates: Set[TransactionState] = Set(Ongoing, PrepareEpochFence)
111117
}
112118

113119
/**
@@ -118,6 +124,7 @@ private[transaction] case object PrepareAbort extends TransactionState {
118124
private[transaction] case object CompleteCommit extends TransactionState {
119125
val id: Byte = 4
120126
val name: String = "CompleteCommit"
127+
val validPreviousStates: Set[TransactionState] = Set(PrepareCommit)
121128
}
122129

123130
/**
@@ -128,6 +135,7 @@ private[transaction] case object CompleteCommit extends TransactionState {
128135
private[transaction] case object CompleteAbort extends TransactionState {
129136
val id: Byte = 5
130137
val name: String = "CompleteAbort"
138+
val validPreviousStates: Set[TransactionState] = Set(PrepareAbort)
131139
}
132140

133141
/**
@@ -136,6 +144,7 @@ private[transaction] case object CompleteAbort extends TransactionState {
136144
private[transaction] case object Dead extends TransactionState {
137145
val id: Byte = 6
138146
val name: String = "Dead"
147+
val validPreviousStates: Set[TransactionState] = Set(Empty, CompleteAbort, CompleteCommit)
139148
}
140149

141150
/**
@@ -145,6 +154,7 @@ private[transaction] case object Dead extends TransactionState {
145154
private[transaction] case object PrepareEpochFence extends TransactionState {
146155
val id: Byte = 7
147156
val name: String = "PrepareEpochFence"
157+
val validPreviousStates: Set[TransactionState] = Set(Ongoing)
148158
}
149159

150160
private[transaction] object TransactionMetadata {
@@ -162,20 +172,6 @@ private[transaction] object TransactionMetadata {
162172
new TransactionMetadata(transactionalId, producerId, lastProducerId, producerEpoch, lastProducerEpoch,
163173
txnTimeoutMs, state, collection.mutable.Set.empty[TopicPartition], timestamp, timestamp)
164174

165-
def isValidTransition(oldState: TransactionState, newState: TransactionState): Boolean =
166-
TransactionMetadata.validPreviousStates(newState).contains(oldState)
167-
168-
private val validPreviousStates: Map[TransactionState, Set[TransactionState]] =
169-
Map(Empty -> Set(Empty, CompleteCommit, CompleteAbort),
170-
Ongoing -> Set(Ongoing, Empty, CompleteCommit, CompleteAbort),
171-
PrepareCommit -> Set(Ongoing),
172-
PrepareAbort -> Set(Ongoing, PrepareEpochFence),
173-
CompleteCommit -> Set(PrepareCommit),
174-
CompleteAbort -> Set(PrepareAbort),
175-
Dead -> Set(Empty, CompleteAbort, CompleteCommit),
176-
PrepareEpochFence -> Set(Ongoing)
177-
)
178-
179175
def isEpochExhausted(producerEpoch: Short): Boolean = producerEpoch >= Short.MaxValue - 1
180176
}
181177

@@ -385,7 +381,7 @@ private[transaction] class TransactionMetadata(val transactionalId: String,
385381
throw new IllegalArgumentException(s"Illegal new producer epoch $newEpoch")
386382

387383
// check that the new state transition is valid and update the pending state if necessary
388-
if (TransactionMetadata.validPreviousStates(newState).contains(state)) {
384+
if (newState.validPreviousStates.contains(state)) {
389385
val transitMetadata = TxnTransitMetadata(newProducerId, producerId, newEpoch, newLastEpoch, newTxnTimeoutMs, newState,
390386
newTopicPartitions, newTxnStartTimestamp, updateTimestamp)
391387
debug(s"TransactionalId $transactionalId prepare transition from $state to $transitMetadata")

core/src/main/scala/kafka/raft/RaftManager.scala

Lines changed: 2 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -21,7 +21,6 @@ import java.nio.file.Files
2121
import java.util
2222
import java.util.OptionalInt
2323
import java.util.concurrent.CompletableFuture
24-
2524
import kafka.log.Log
2625
import kafka.raft.KafkaRaftManager.RaftIoThread
2726
import kafka.server.{KafkaConfig, MetaProperties}
@@ -37,8 +36,8 @@ import org.apache.kafka.common.security.JaasContext
3736
import org.apache.kafka.common.security.auth.SecurityProtocol
3837
import org.apache.kafka.common.utils.{LogContext, Time}
3938
import org.apache.kafka.raft.RaftConfig.{AddressSpec, InetAddressSpec, NON_ROUTABLE_ADDRESS, UnknownAddressSpec}
40-
import org.apache.kafka.raft.{FileBasedStateStore, KafkaRaftClient, RaftClient, RaftConfig, RaftRequest, RecordSerde}
41-
39+
import org.apache.kafka.raft.{FileBasedStateStore, KafkaRaftClient, RaftClient, RaftConfig, RaftRequest}
40+
import org.apache.kafka.server.common.serialization.RecordSerde
4241
import scala.jdk.CollectionConverters._
4342

4443
object KafkaRaftManager {

core/src/main/scala/kafka/server/ControllerApis.scala

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -21,7 +21,6 @@ import java.util
2121
import java.util.Collections
2222
import java.util.Map.Entry
2323
import java.util.concurrent.{CompletableFuture, ExecutionException}
24-
2524
import kafka.network.RequestChannel
2625
import kafka.raft.RaftManager
2726
import kafka.server.QuotaFactory.QuotaManagers
@@ -50,8 +49,9 @@ import org.apache.kafka.common.resource.ResourceType.{CLUSTER, TOPIC}
5049
import org.apache.kafka.common.utils.Time
5150
import org.apache.kafka.common.{Node, Uuid}
5251
import org.apache.kafka.controller.Controller
53-
import org.apache.kafka.metadata.{ApiMessageAndVersion, BrokerHeartbeatReply, BrokerRegistrationReply, VersionRange}
52+
import org.apache.kafka.metadata.{BrokerHeartbeatReply, BrokerRegistrationReply, VersionRange}
5453
import org.apache.kafka.server.authorizer.Authorizer
54+
import org.apache.kafka.server.common.ApiMessageAndVersion
5555

5656
import scala.jdk.CollectionConverters._
5757

core/src/main/scala/kafka/server/ControllerServer.scala

Lines changed: 2 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -20,7 +20,6 @@ package kafka.server
2020
import java.util.concurrent.{CompletableFuture, TimeUnit}
2121
import java.util
2222
import java.util.concurrent.locks.ReentrantLock
23-
2423
import kafka.cluster.Broker.ServerInfo
2524
import kafka.log.LogConfig
2625
import kafka.metrics.{KafkaMetricsGroup, KafkaYammerMetrics, LinuxIoMetricsCollector}
@@ -37,11 +36,12 @@ import org.apache.kafka.common.security.token.delegation.internals.DelegationTok
3736
import org.apache.kafka.common.utils.{LogContext, Time}
3837
import org.apache.kafka.common.{ClusterResource, Endpoint}
3938
import org.apache.kafka.controller.{Controller, QuorumController, QuorumControllerMetrics}
40-
import org.apache.kafka.metadata.{ApiMessageAndVersion, VersionRange}
39+
import org.apache.kafka.metadata.VersionRange
4140
import org.apache.kafka.metalog.MetaLogManager
4241
import org.apache.kafka.raft.RaftConfig
4342
import org.apache.kafka.raft.RaftConfig.AddressSpec
4443
import org.apache.kafka.server.authorizer.Authorizer
44+
import org.apache.kafka.server.common.ApiMessageAndVersion
4545

4646
import scala.jdk.CollectionConverters._
4747

core/src/main/scala/kafka/server/KafkaRaftServer.scala

Lines changed: 1 addition & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -18,7 +18,6 @@ package kafka.server
1818

1919
import java.io.File
2020
import java.util.concurrent.CompletableFuture
21-
2221
import kafka.common.{InconsistentNodeIdException, KafkaException}
2322
import kafka.log.Log
2423
import kafka.metrics.{KafkaMetricsReporter, KafkaYammerMetrics}
@@ -27,9 +26,9 @@ import kafka.server.KafkaRaftServer.{BrokerRole, ControllerRole}
2726
import kafka.utils.{CoreUtils, Logging, Mx4jLoader, VerifiableProperties}
2827
import org.apache.kafka.common.{TopicPartition, Uuid}
2928
import org.apache.kafka.common.utils.{AppInfoParser, Time}
30-
import org.apache.kafka.metadata.ApiMessageAndVersion
3129
import org.apache.kafka.raft.RaftConfig
3230
import org.apache.kafka.raft.metadata.{MetaLogRaftShim, MetadataRecordSerde}
31+
import org.apache.kafka.server.common.ApiMessageAndVersion
3332

3433
import scala.collection.Seq
3534

core/src/main/scala/kafka/tools/TestRaftServer.scala

Lines changed: 2 additions & 1 deletion
Original file line numberDiff line numberDiff line change
@@ -35,7 +35,8 @@ import org.apache.kafka.common.security.scram.internals.ScramMechanism
3535
import org.apache.kafka.common.security.token.delegation.internals.DelegationTokenCache
3636
import org.apache.kafka.common.utils.{Time, Utils}
3737
import org.apache.kafka.common.{TopicPartition, Uuid, protocol}
38-
import org.apache.kafka.raft.{Batch, BatchReader, RaftClient, RaftConfig, RecordSerde}
38+
import org.apache.kafka.raft.{Batch, BatchReader, RaftClient, RaftConfig}
39+
import org.apache.kafka.server.common.serialization.RecordSerde
3940
import org.apache.kafka.snapshot.SnapshotReader
4041

4142
import scala.jdk.CollectionConverters._

0 commit comments

Comments
 (0)