[ https://issues.apache.org/jira/browse/FLINK-10166?page=com.atlassian.jira.plugin.system.issuetabpanels:comment-tabpanel&focusedCommentId=16669806#comment-16669806 ]
ASF GitHub Bot commented on FLINK-10166: ---------------------------------------- twalthr closed pull request #6863: [FLINK-10166][table] Replace commons.codec.binary.Base64 with java.util.Base64 URL: https://github.com/apache/flink/pull/6863 This is a PR merged from a forked repository. As GitHub hides the original diff on merge, it is displayed below for the sake of provenance: As this is a foreign pull request (from a fork), the diff is supplied below (as it won't show otherwise due to GitHub magic): diff --git a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/AggregationCodeGenerator.scala b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/AggregationCodeGenerator.scala index 11c00080329..c7a6a64ab94 100644 --- a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/AggregationCodeGenerator.scala +++ b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/codegen/AggregationCodeGenerator.scala @@ -21,7 +21,6 @@ import java.lang.reflect.Modifier import java.lang.{Iterable => JIterable} import org.apache.calcite.rex.RexLiteral -import org.apache.commons.codec.binary.Base64 import org.apache.flink.api.common.state.{ListStateDescriptor, MapStateDescriptor, State, StateDescriptor} import org.apache.flink.api.common.typeinfo.{BasicTypeInfo, TypeInformation} import org.apache.flink.api.java.typeutils.RowTypeInfo @@ -330,7 +329,7 @@ class AggregationCodeGenerator( s""" | $descClassQualifier $descFieldTerm = ($descClassQualifier) | org.apache.flink.util.InstantiationUtil.deserializeObject( - | org.apache.commons.codec.binary.Base64.decodeBase64("$serializedData"), + | java.util.Base64.getUrlDecoder().decode("$serializedData".getBytes("UTF-8")), | $contextTerm.getUserCodeClassLoader()); |""".stripMargin val createDataView = if (dataViewField.getType == classOf[MapView[_, _]]) { @@ -774,6 +773,6 @@ class AggregationCodeGenerator( @throws[Exception] def serializeStateDescriptor(stateDescriptor: StateDescriptor[_, _]): String = { val byteArray = InstantiationUtil.serializeObject(stateDescriptor) - Base64.encodeBase64URLSafeString(byteArray) + new String(java.util.Base64.getUrlEncoder.withoutPadding().encode(byteArray), "UTF-8") } } diff --git a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/descriptors/DescriptorProperties.scala b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/descriptors/DescriptorProperties.scala index 9328c806913..4177c3a1620 100644 --- a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/descriptors/DescriptorProperties.scala +++ b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/descriptors/DescriptorProperties.scala @@ -26,7 +26,6 @@ import java.util.function.{Consumer, Supplier} import java.util.regex.Pattern import java.util.{Optional, List => JList, Map => JMap} -import org.apache.commons.codec.binary.Base64 import org.apache.commons.lang.StringEscapeUtils import org.apache.flink.api.common.typeinfo.TypeInformation import org.apache.flink.api.java.tuple.{Tuple2 => JTuple2} @@ -1356,7 +1355,7 @@ object DescriptorProperties { } try { val byteArray = InstantiationUtil.serializeObject(obj) - Base64.encodeBase64URLSafeString(byteArray) + new String(java.util.Base64.getUrlEncoder.withoutPadding().encode(byteArray), "UTF-8") } catch { case e: Exception => throw new ValidationException( @@ -1366,7 +1365,7 @@ object DescriptorProperties { def deserialize[T](data: String, expected: Class[T]): T = { try { - val byteData = Base64.decodeBase64(data) + val byteData = java.util.Base64.getUrlDecoder.decode(data.getBytes("UTF-8")) val obj = InstantiationUtil.deserializeObject[T]( byteData, Thread.currentThread.getContextClassLoader) diff --git a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/functions/utils/UserDefinedFunctionUtils.scala b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/functions/utils/UserDefinedFunctionUtils.scala index 2c08001d20c..7e5128cf939 100644 --- a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/functions/utils/UserDefinedFunctionUtils.scala +++ b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/functions/utils/UserDefinedFunctionUtils.scala @@ -24,7 +24,6 @@ import java.lang.{Integer => JInt, Long => JLong} import java.lang.reflect.{Method, Modifier} import java.sql.{Date, Time, Timestamp} -import org.apache.commons.codec.binary.Base64 import com.google.common.primitives.Primitives import org.apache.calcite.rel.`type`.RelDataType import org.apache.calcite.sql.`type`.SqlOperandTypeChecker.Consistency @@ -736,12 +735,12 @@ object UserDefinedFunctionUtils { @throws[Exception] def serialize(function: UserDefinedFunction): String = { val byteArray = InstantiationUtil.serializeObject(function) - Base64.encodeBase64URLSafeString(byteArray) + new String(java.util.Base64.getUrlEncoder.withoutPadding().encode(byteArray), "UTF-8") } @throws[Exception] def deserialize(data: String): UserDefinedFunction = { - val byteData = Base64.decodeBase64(data) + val byteData = java.util.Base64.getUrlDecoder.decode(data.getBytes("UTF-8")) InstantiationUtil .deserializeObject[UserDefinedFunction](byteData, Thread.currentThread.getContextClassLoader) } diff --git a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/runtime/functions/ScalarFunctions.scala b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/runtime/functions/ScalarFunctions.scala index 1db4da2f899..c1fe846f8c0 100644 --- a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/runtime/functions/ScalarFunctions.scala +++ b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/runtime/functions/ScalarFunctions.scala @@ -23,7 +23,7 @@ import java.nio.charset.StandardCharsets import java.util.regex.Matcher import java.util.regex.Pattern -import org.apache.commons.codec.binary.{Base64, Hex} +import org.apache.commons.codec.binary.Hex import org.apache.commons.lang3.StringUtils import scala.annotation.varargs @@ -256,13 +256,15 @@ object ScalarFunctions { * Returns the base string decoded with base64. */ def fromBase64(str: String): String = - new String(Base64.decodeBase64(str), StandardCharsets.UTF_8) + new String(java.util.Base64.getDecoder.decode(str.getBytes("UTF-8")), "UTF-8") /** * Returns the base64-encoded result of the input string. */ def toBase64(base: String): String = - Base64.encodeBase64String(base.getBytes(StandardCharsets.UTF_8)) + new String( + java.util.Base64.getEncoder.encode(base.getBytes("UTF-8")), + "UTF-8") /** * Returns the hex string of a long argument. diff --git a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/typeutils/TypeStringUtils.scala b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/typeutils/TypeStringUtils.scala index 9e5f075cd4b..264ab4ee77e 100644 --- a/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/typeutils/TypeStringUtils.scala +++ b/flink-libraries/flink-table/src/main/scala/org/apache/flink/table/typeutils/TypeStringUtils.scala @@ -20,7 +20,6 @@ package org.apache.flink.table.typeutils import java.io.Serializable -import org.apache.commons.codec.binary.Base64 import org.apache.commons.lang3.StringEscapeUtils import org.apache.flink.api.common.functions.InvalidTypesException import org.apache.flink.api.common.typeinfo.{PrimitiveArrayTypeInfo, TypeInformation} @@ -267,7 +266,7 @@ object TypeStringUtils extends JavaTokenParsers with PackratParsers { private def serialize(obj: Serializable): String = { try { val byteArray = InstantiationUtil.serializeObject(obj) - Base64.encodeBase64URLSafeString(byteArray) + new String(java.util.Base64.getUrlEncoder.withoutPadding().encode(byteArray), "UTF-8") } catch { case e: Exception => throw new ValidationException(s"Unable to serialize type information '$obj' with " + @@ -276,7 +275,7 @@ object TypeStringUtils extends JavaTokenParsers with PackratParsers { } private def deserialize(data: String): TypeInformation[_] = { - val byteData = Base64.decodeBase64(data) + val byteData = java.util.Base64.getUrlDecoder.decode(data.getBytes("UTF-8")) InstantiationUtil .deserializeObject[TypeInformation[_]](byteData, Thread.currentThread.getContextClassLoader) } diff --git a/flink-runtime/src/test/java/org/apache/flink/runtime/util/Base64CompatibleTest.java b/flink-runtime/src/test/java/org/apache/flink/runtime/util/Base64CompatibleTest.java new file mode 100644 index 00000000000..c437028cb84 --- /dev/null +++ b/flink-runtime/src/test/java/org/apache/flink/runtime/util/Base64CompatibleTest.java @@ -0,0 +1,98 @@ +/* + * 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.flink.runtime.util; + +import org.junit.Test; + +import static org.junit.Assert.assertEquals; +import static org.junit.Assert.assertTrue; + +/** + * Base64 Library switch from org.apache.commons.codec.binary.Base64 to java.util.Base64. + * Test cases provided to make sure Base64 content compatible. + */ +public class Base64CompatibleTest +{ + @Test + public void testCommonsBase64ToJavaUtilBase64() throws Exception + { + String inputString = "Hello, this is apache flink."; + String encodeString = + org.apache.commons.codec.binary.Base64.encodeBase64URLSafeString( + inputString.getBytes("UTF-8") + ); + + String decodeString = + new String( + java.util.Base64.getUrlDecoder().decode(encodeString), + "UTF-8" + ); + + assertEquals(inputString, decodeString); + } + + + /** + * encodeBase64URLSafeString with inputString will have char '_' in decodeString. + * must use RFC4648_URLSAFE to decode. + * @throws Exception + */ + @Test(expected = IllegalArgumentException.class) + public void testCommonsBase64ToJavaUtilBase64UsingWrongDecoder1() throws Exception + { + String inputString = "special base64 char <_> 00?"; + String encodeString = + org.apache.commons.codec.binary.Base64.encodeBase64URLSafeString( + inputString.getBytes("UTF-8") + ); + + assertTrue(encodeString.contains("_")); + + String decodeString = + new String( + java.util.Base64.getDecoder().decode(encodeString), + "UTF-8" + ); + + } + + /** + * encodeBase64URLSafeString with inputString will have char '-' in decodeString. + * must use RFC4648_URLSAFE to decode. + * @throws Exception + */ + @Test(expected = IllegalArgumentException.class) + public void testCommonsBase64ToJavaUtilBase64UsingWrongDecoder2() throws Exception + { + String inputString = "special base64 char <-> 00>"; + String encodeString = + org.apache.commons.codec.binary.Base64.encodeBase64URLSafeString( + inputString.getBytes("UTF-8") + ); + + assertTrue(encodeString.contains("-")); + + String decodeString = + new String( + java.util.Base64.getDecoder().decode(encodeString), + "UTF-8" + ); + + } +} ---------------------------------------------------------------- This is an automated message from the Apache Git Service. To respond to the message, please log on GitHub and use the URL above to go to the specific comment. For queries about this service, please contact Infrastructure at: us...@infra.apache.org > Dependency problems when executing SQL query in sql-client > ---------------------------------------------------------- > > Key: FLINK-10166 > URL: https://issues.apache.org/jira/browse/FLINK-10166 > Project: Flink > Issue Type: Bug > Components: Table API & SQL > Affects Versions: 1.6.0 > Reporter: Dawid Wysakowicz > Assignee: Timo Walther > Priority: Blocker > Labels: pull-request-available > Fix For: 1.7.0 > > > When tried to run query: > {code} > select count(distinct name) from (Values ('a'), ('b')) AS NameTable(name) > {code} > in {{sql-client.sh}} I got: > {code} > [ERROR] Could not execute SQL statement. Reason: > org.codehaus.commons.compiler.CompileException: Line 43, Column 10: Unknown > variable or type "org.apache.commons.codec.binary.Base64" > {code} -- This message was sent by Atlassian JIRA (v7.6.3#76005)