Skip to content

Commit e81bac4

Browse files
RyanSkrabaMartijnVisser
authored andcommitted
[FLINK-20625][pubsub] Align new source with Kafka package names
1 parent 81775b0 commit e81bac4

File tree

14 files changed

+59
-48
lines changed

14 files changed

+59
-48
lines changed
File renamed without changes.
Lines changed: 11 additions & 10 deletions
Original file line numberDiff line numberDiff line change
@@ -7,16 +7,17 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source;
20+
package org.apache.flink.connector.gcp.pubsub.source;
2021

2122
import org.apache.flink.api.common.serialization.DeserializationSchema;
2223
import org.apache.flink.api.common.typeinfo.TypeInformation;
@@ -31,19 +32,19 @@
3132
import org.apache.flink.configuration.Configuration;
3233
import org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
3334
import org.apache.flink.connector.base.source.reader.synchronization.FutureCompletingBlockingQueue;
35+
import org.apache.flink.connector.gcp.pubsub.source.enumerator.PubSubEnumeratorState;
36+
import org.apache.flink.connector.gcp.pubsub.source.enumerator.PubSubEnumeratorStateSerializer;
37+
import org.apache.flink.connector.gcp.pubsub.source.enumerator.PubSubSourceEnumerator;
38+
import org.apache.flink.connector.gcp.pubsub.source.reader.PubSubRecordEmitter;
39+
import org.apache.flink.connector.gcp.pubsub.source.reader.PubSubSourceReader;
40+
import org.apache.flink.connector.gcp.pubsub.source.reader.PubSubSplitReader;
41+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplit;
42+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplitSerializer;
3443
import org.apache.flink.core.io.SimpleVersionedSerializer;
3544
import org.apache.flink.streaming.connectors.gcp.pubsub.DefaultPubSubSubscriberFactory;
3645
import org.apache.flink.streaming.connectors.gcp.pubsub.DeserializationSchemaWrapper;
3746
import org.apache.flink.streaming.connectors.gcp.pubsub.common.PubSubDeserializationSchema;
3847
import org.apache.flink.streaming.connectors.gcp.pubsub.common.PubSubSubscriberFactory;
39-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.enumerator.PubSubEnumeratorState;
40-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.enumerator.PubSubEnumeratorStateSerializer;
41-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.enumerator.PubSubSourceEnumerator;
42-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.reader.PubSubRecordEmitter;
43-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.reader.PubSubSourceReader;
44-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.reader.PubSubSplitReader;
45-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplit;
46-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplitSerializer;
4748
import org.apache.flink.util.Preconditions;
4849

4950
import com.google.auth.Credentials;
Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -7,16 +7,17 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.enumerator;
20+
package org.apache.flink.connector.gcp.pubsub.source.enumerator;
2021

2122
/**
2223
* A stub to contain the checkpoint data of a {@link PubSubSourceEnumerator}. GCP Pub/Sub does not
Lines changed: 3 additions & 2 deletions
Original file line numberDiff line numberDiff line change
@@ -7,16 +7,17 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.enumerator;
20+
package org.apache.flink.connector.gcp.pubsub.source.enumerator;
2021

2122
import org.apache.flink.core.io.SimpleVersionedSerializer;
2223

Lines changed: 7 additions & 7 deletions
Original file line numberDiff line numberDiff line change
@@ -7,31 +7,31 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.enumerator;
20+
package org.apache.flink.connector.gcp.pubsub.source.enumerator;
2021

2122
import org.apache.flink.api.connector.source.SplitEnumerator;
2223
import org.apache.flink.api.connector.source.SplitEnumeratorContext;
23-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplit;
24+
import org.apache.flink.connector.gcp.pubsub.source.PubSubSource;
25+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplit;
2426

2527
import javax.annotation.Nullable;
2628

2729
import java.io.IOException;
2830
import java.util.List;
2931

3032
/**
31-
* The enumerator for the {@link
32-
* org.apache.flink.streaming.connectors.gcp.pubsub.source.PubSubSource}. It does not do any work
33-
* discovery as envisioned by FLIP-27 because GCP Pub/Sub hides partitions and other implementation
34-
* details.
33+
* The enumerator for the {@link PubSubSource}. It does not do any work discovery as envisioned by
34+
* FLIP-27 because GCP Pub/Sub hides partitions and other implementation details.
3535
*/
3636
public class PubSubSourceEnumerator implements SplitEnumerator<PubSubSplit, PubSubEnumeratorState> {
3737
private final SplitEnumeratorContext<PubSubSplit> context;
Lines changed: 4 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -7,21 +7,22 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.reader;
20+
package org.apache.flink.connector.gcp.pubsub.source.reader;
2021

2122
import org.apache.flink.api.connector.source.SourceOutput;
2223
import org.apache.flink.api.java.tuple.Tuple2;
2324
import org.apache.flink.connector.base.source.reader.RecordEmitter;
24-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplitState;
25+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplitState;
2526

2627
/**
2728
* A custom {@link RecordEmitter} to emit a record which includes the data of the received GCP
Lines changed: 4 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -7,16 +7,17 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.reader;
20+
package org.apache.flink.connector.gcp.pubsub.source.reader;
2021

2122
import org.apache.flink.api.java.tuple.Tuple2;
2223
import org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
@@ -25,7 +26,7 @@
2526
import org.apache.flink.connector.base.source.reader.fetcher.SplitFetcherTask;
2627
import org.apache.flink.connector.base.source.reader.splitreader.SplitReader;
2728
import org.apache.flink.connector.base.source.reader.synchronization.FutureCompletingBlockingQueue;
28-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplit;
29+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplit;
2930

3031
import org.slf4j.Logger;
3132
import org.slf4j.LoggerFactory;
Lines changed: 5 additions & 4 deletions
Original file line numberDiff line numberDiff line change
@@ -7,16 +7,17 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.reader;
20+
package org.apache.flink.connector.gcp.pubsub.source.reader;
2021

2122
import org.apache.flink.api.connector.source.SourceReaderContext;
2223
import org.apache.flink.api.java.tuple.Tuple2;
@@ -25,8 +26,8 @@
2526
import org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
2627
import org.apache.flink.connector.base.source.reader.SingleThreadMultiplexSourceReaderBase;
2728
import org.apache.flink.connector.base.source.reader.synchronization.FutureCompletingBlockingQueue;
28-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplit;
29-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplitState;
29+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplit;
30+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplitState;
3031

3132
import org.slf4j.Logger;
3233
import org.slf4j.LoggerFactory;
Lines changed: 4 additions & 3 deletions
Original file line numberDiff line numberDiff line change
@@ -7,26 +7,27 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.reader;
20+
package org.apache.flink.connector.gcp.pubsub.source.reader;
2021

2122
import org.apache.flink.api.java.tuple.Tuple2;
2223
import org.apache.flink.connector.base.source.reader.RecordsBySplits;
2324
import org.apache.flink.connector.base.source.reader.RecordsWithSplitIds;
2425
import org.apache.flink.connector.base.source.reader.splitreader.SplitReader;
2526
import org.apache.flink.connector.base.source.reader.splitreader.SplitsChange;
27+
import org.apache.flink.connector.gcp.pubsub.source.split.PubSubSplit;
2628
import org.apache.flink.streaming.connectors.gcp.pubsub.common.PubSubDeserializationSchema;
2729
import org.apache.flink.streaming.connectors.gcp.pubsub.common.PubSubSubscriber;
2830
import org.apache.flink.streaming.connectors.gcp.pubsub.common.PubSubSubscriberFactory;
29-
import org.apache.flink.streaming.connectors.gcp.pubsub.source.split.PubSubSplit;
3031
import org.apache.flink.util.Collector;
3132

3233
import com.google.auth.Credentials;
Lines changed: 6 additions & 5 deletions
Original file line numberDiff line numberDiff line change
@@ -7,23 +7,24 @@
77
* "License"); you may not use this file except in compliance
88
* with the License. You may obtain a copy of the License at
99
*
10-
* http://www.apache.org/licenses/LICENSE-2.0
10+
* https://www.apache.org/licenses/LICENSE-2.0
1111
*
1212
* Unless required by applicable law or agreed to in writing, software
1313
* distributed under the License is distributed on an "AS IS" BASIS,
1414
* WITHOUT WARRANTIES OR CONDITIONS OF ANY KIND, either express or implied.
1515
* See the License for the specific language governing permissions and
1616
* limitations under the License.
17+
*
1718
*/
1819

19-
package org.apache.flink.streaming.connectors.gcp.pubsub.source.split;
20+
package org.apache.flink.connector.gcp.pubsub.source.split;
2021

2122
import org.apache.flink.api.connector.source.SourceSplit;
23+
import org.apache.flink.connector.gcp.pubsub.source.PubSubSource;
2224

2325
/**
24-
* A {@link SourceSplit} implementation for the {@link
25-
* org.apache.flink.streaming.connectors.gcp.pubsub.source.PubSubSource}. It only saves a generic
26-
* split ID because a split does not carry any specific information for pulling from a project
26+
* A {@link SourceSplit} implementation for the {@link PubSubSource}. It only saves a generic split
27+
* ID because a split does not carry any specific information for pulling from a project
2728
* subscription. That is because GCP Pub/Sub hides partitions and other implementation details.
2829
*/
2930
public class PubSubSplit implements SourceSplit {

0 commit comments

Comments
 (0)