KAFKA-7612: Fix javac warnings and enable warnings as errors#5900
KAFKA-7612: Fix javac warnings and enable warnings as errors#5900ijuma merged 34 commits intoapache:trunkfrom
Conversation
There was a problem hiding this comment.
@rhauch Can you please review the Connect changes that start from this point?
There was a problem hiding this comment.
@mjsax @guozhangwang Can you please review the Streams changes that start from this point.
There was a problem hiding this comment.
\cc @bbejeck Can you have a look here? You know the optimizer code best.
There was a problem hiding this comment.
@ijuma I looked into this in more detail. We don't need a new type R -- instead, we should update to
public class KTableKTableJoinNode<K, V1, V2, VR> extends BaseJoinProcessorNode<K, Change<V1>, Change<V2>, Change<VR>>
and use private final MaterializedInternal<K, VR, KeyValueStore<Bytes, byte[]>> materializedInternal; (ie, VR instead of R. (Note, the you always pass <..., Change<X>, X> in the current PR, what is redundant and can be avoided.)
There was a problem hiding this comment.
@mjsax That doesn't work, it's what I tried first. As far as I can tell, the existing code was wrong but it doesn't matter due to the way generics are used here.
There was a problem hiding this comment.
Synced offline, I had misunderstood the suggestion.
There was a problem hiding this comment.
@mjsax Good suggestion, I've added a commit. Let me know what you think.
There was a problem hiding this comment.
the changes to the optimizer code LGTM
mjsax
left a comment
There was a problem hiding this comment.
As far as I can tell, LGTM.
Not sure about the Scala change and why we need to additional generic type though.
a8e6887 to
be7b014
Compare
…eprecated overload
870081c to
797ec97
Compare
|
@omkreddy Do you have time to review this? |
omkreddy
left a comment
There was a problem hiding this comment.
core and clients module changes are looks good to me.
|
JDK 11 build passed and JDK 8 failures are unrelated. @mjsax, can you please take a look at the Connect changes? They're pretty trivial and that would give us enough committer coverage to get this merged before it goes stale. :) |
| private val uncleanablePartitions = mutable.HashMap[String, mutable.Set[TopicPartition]]() | ||
|
|
||
| /* the set of directories marked as uncleanable and therefore offline */ | ||
| private val uncleanableDirs = mutable.HashSet[String]() |
There was a problem hiding this comment.
looks good. thanks
| // Visible for testing | ||
| private[server] def getFetcher(topicPartition: TopicPartition): Option[T] = { | ||
| lock synchronized { | ||
| val fetcherId = getFetcherId(topicPartition) |
rhauch
left a comment
There was a problem hiding this comment.
Connect changes look good to me, tho I'm not a committer.
7aa0320 to
7d2c25b
Compare
|
Only testCoordinatorFailover failed and it's unrelated (there's a separate PR to fix that). I'll go ahead and merge this. If there are any additional comments, I'll follow-up in a subsequent PR. |
guozhangwang
left a comment
There was a problem hiding this comment.
Made a pass over streams part. LGTM except one question (above).
I've also made a minor PR for replacing some deprecated functions, maybe better merging this one and rebase:
| return viewByRegion; | ||
| }) | ||
| .map((user, viewRegion) -> new KeyValue<>(viewRegion.region, viewRegion)) | ||
| .groupByKey(Serialized.with(Serdes.String(), new JSONSerde<>())) |
There was a problem hiding this comment.
We can replace Serialized with Grouped, ditto below.
cc @bbejeck
| @Deprecated | ||
| public <KR> KGroupedStream<KR, V> groupBy(final KeyValueMapper<? super K, ? super V, KR> selector, | ||
| final Serialized<KR, V> serialized) { | ||
| final org.apache.kafka.streams.kstream.Serialized<KR, V> serialized) { |
| * Too much specific information to generalize so the KTable-KTable join requires a specific node. | ||
| */ | ||
| public class KTableKTableJoinNode<K, V1, V2, VR> extends BaseJoinProcessorNode<K, V1, V2, VR> { | ||
| public class KTableKTableJoinNode<K, V1, V2, VR> extends BaseJoinProcessorNode<K, Change<V1>, Change<V2>, Change<VR>> { |
There was a problem hiding this comment.
What's the purpose of moving Change<V> inside this class instead of on the callers?
There was a problem hiding this comment.
That was @mjsax suggestion to avoid adding an additional type parameter for Materialized. See the last commit in the PR.
| .groupBy( | ||
| mapper, | ||
| Serialized.with(Serdes.String(), Serdes.String())); | ||
| org.apache.kafka.streams.kstream.Serialized.with(Serdes.String(), Serdes.String())); |
There was a problem hiding this comment.
Ditto here and below: use Grouped instead of Serialized.
| .groupBy( | ||
| mapper, | ||
| Serialized.with(Serdes.String(), Serdes.String())); | ||
| org.apache.kafka.streams.kstream.Serialized.with(Serdes.String(), Serdes.String())); |
There was a problem hiding this comment.
Ditto here and below.
| import static org.junit.Assert.fail; | ||
|
|
||
|
|
||
| @SuppressWarnings("deprecation") |
There was a problem hiding this comment.
We should replace until with grace here, cc @vvcephei
| import static org.junit.Assert.assertTrue; | ||
| import static org.junit.Assert.fail; | ||
|
|
||
| @SuppressWarnings("deprecation") |
There was a problem hiding this comment.
We should replace of with the other overloaded function. cc @vvcephei
|
@guozhangwang Thanks for the review. I responded to the question about the type parameters. All the comments about removing deprecated usage with the undeprecated alternative seem to be addressed in your follow-up PR so we can just discuss it there. This one was already merged |
|
Yeah I guess it was merged while I was preparing the PR, and I did not notice it after rebasing trunk :) Sounds good. |
…5900) - Use Xlint:all with 3 exclusions (filed KAFKA-7613 to remove the exclusions) - Use the same javac options when compiling tests (seems accidental that we didn't do this before) - Replaced several deprecated method calls with non-deprecated ones: - `KafkaConsumer.poll(long)` and `KafkaConsumer.close(long)` - `Class.newInstance` and `new Integer/Long` (deprecated since Java 9) - `scala.Console` (deprecated in Scala 2.11) - `PartitionData` taking a timestamp (one of them seemingly a bug) - `JsonMappingException` single parameter constructor - Fix unnecessary usage of raw types in several places. - Add @SuppressWarnings for deprecations, unchecked and switch fallthrough in several places. - Scala clean-ups (var -> val, ETA expansion warnings, avoid reflective calls) - Use lambdas to simplify code in a few places - Add @SafeVarargs, fix varargs usage and remove unnecessary `Utils.mkList` method Reviewers: Matthias J. Sax <mjsax@apache.org>, Manikumar Reddy <manikumar.reddy@gmail.com>, Randall Hauch <rhauch@gmail.com>, Bill Bejeck <bill@confluent.io>, Stanislav Kozlovski <stanislav_kozlovski@outlook.com>
we didn't do this before)
KafkaConsumer.poll(long)andKafkaConsumer.close(long)Class.newInstanceandnew Integer/Long(deprecated since Java 9)scala.Console(deprecated in Scala 2.11)PartitionDatataking a timestamp (one of them seemingly a bug)JsonMappingExceptionsingle parameter constructorseveral places.
Utils.mkListmethodCommitter Checklist (excluded from commit message)